apache_beam.transforms.periodicsequence module

class apache_beam.transforms.periodicsequence.ImpulseSeqGenRestrictionProvider[source]

Bases: apache_beam.transforms.core.RestrictionProvider

initial_restriction(element)[source]
create_tracker(restriction)[source]
restriction_size(unused_element, restriction)[source]
class apache_beam.transforms.periodicsequence.ImpulseSeqGenDoFn(*unused_args, **unused_kwargs)[source]

Bases: apache_beam.transforms.core.DoFn

ImpulseSeqGenDoFn fn receives tuple elements with three parts:

  • first_timestamp = first timestamp to output element for.
  • last_timestamp = last timestamp/time to output element for.
  • fire_interval = how often to fire an element.

For each input element received, ImpulseSeqGenDoFn fn will start generating output elements in following pattern:

  • if element timestamp is less than current runtime then output element.
  • if element timestamp is greater than current runtime, wait until next element timestamp.

ImpulseSeqGenDoFn can’t guarantee that each element is output at exact time. ImpulseSeqGenDoFn guarantees that elements would not be output prior to given runtime timestamp.

process(element, restriction_tracker=RestrictionParam(ImpulseSeqGenRestrictionProvider))[source]
Parameters:
  • element – (start_timestamp, end_timestamp, interval)
  • restriction_tracker
Returns:

yields elements at processing real-time intervals with value of target output timestamp for the element.

class apache_beam.transforms.periodicsequence.PeriodicSequence(label=None)[source]

Bases: apache_beam.transforms.ptransform.PTransform

PeriodicSequence transform receives tuple elements with three parts:

  • first_timestamp = first timestamp to output element for.
  • last_timestamp = last timestamp/time to output element for.
  • fire_interval = how often to fire an element.

For each input element received, PeriodicSequence transform will start generating output elements in following pattern:

  • if element timestamp is less than current runtime then output element.
  • if element timestamp is greater than current runtime, wait until next element timestamp.

PeriodicSequence can’t guarantee that each element is output at exact time. PeriodicSequence guarantees that elements would not be output prior to given runtime timestamp.

expand(pcoll)[source]
class apache_beam.transforms.periodicsequence.PeriodicImpulse(start_timestamp=Timestamp(1607392201.338522), stop_timestamp=Timestamp(9223372036854.775000), fire_interval=360.0, apply_windowing=False)[source]

Bases: apache_beam.transforms.ptransform.PTransform

PeriodicImpulse transform generates an infinite sequence of elements with given runtime interval.

PeriodicImpulse transform behaves same as {@link PeriodicSequence} transform, but can be used as first transform in pipeline.

Parameters:
  • start_timestamp – Timestamp for first element.
  • stop_timestamp – Timestamp after which no elements will be output.
  • fire_interval – Interval at which to output elements.
  • apply_windowing – Whether each element should be assigned to individual window. If false, all elements will reside in global window.
expand(pbegin)[source]