Enum Class PaneInfo.Timing

java.lang.Object
java.lang.Enum<PaneInfo.Timing>
org.apache.beam.sdk.transforms.windowing.PaneInfo.Timing
All Implemented Interfaces:
Serializable, Comparable<PaneInfo.Timing>, Constable
Enclosing class:
PaneInfo

public static enum PaneInfo.Timing extends Enum<PaneInfo.Timing>
Enumerates the possibilities for the timing of this pane firing related to the input and output watermarks for its computation.

A window may fire multiple panes, and the timing of those panes generally follows the regular expression EARLY* ON_TIME? LATE*. Generally a pane is considered:

  1. EARLY if the system cannot be sure it has seen all data which may contribute to the pane's window.
  2. ON_TIME if the system predicts it has seen all the data which may contribute to the pane's window.
  3. LATE if the system has encountered new data after predicting no more could arrive. It is possible an ON_TIME pane has already been emitted, in which case any following panes are considered LATE.

Only an AfterWatermark.pastEndOfWindow() trigger may produce an ON_TIME pane. With merging WindowFn's, windows may be merged to produce new windows that satisfy their own instance of the above regular expression. The only guarantee is that once a window produces a final pane, it will not be merged into any new windows.

The predictions above are made using the mechanism of watermarks.

We can state some properties of LATE and ON_TIME panes, but first need some definitions:

  1. We'll call a pipeline 'simple' if it does not use DoFn.WindowedContext.outputWithTimestamp(OutputT, org.joda.time.Instant) in any DoFn, and it uses the same Window.withAllowedLateness(org.joda.time.Duration) argument value on all windows (or uses the default of Duration.ZERO).
  2. We'll call an element 'locally late', from the point of view of a computation on a worker, if the element's timestamp is before the input watermark for that computation on that worker. The element is otherwise 'locally on-time'.
  3. We'll say 'the pane's timestamp' to mean the timestamp of the element produced to represent the pane's contents.

Then in simple pipelines:

  1. (Soundness) An ON_TIME pane can never cause a later computation to generate a LATE pane. (If it did, it would imply a later computation's input watermark progressed ahead of an earlier stage's output watermark, which by design is not possible.)
  2. (Liveness) An ON_TIME pane is emitted as soon as possible after the input watermark passes the end of the pane's window.
  3. (Consistency) A pane with only locally on-time elements will always be ON_TIME. And a LATE pane cannot contain locally on-time elements.

However, note that:

  1. An ON_TIME pane may contain locally late elements. It may even contain only locally late elements. Provided a locally late element finds its way into an ON_TIME pane its lateness becomes unobservable.
  2. A LATE pane does not necessarily cause any following computation panes to be marked as LATE.
  • Enum Constant Details

    • EARLY

      public static final PaneInfo.Timing EARLY
      Pane was fired before the input watermark had progressed after the end of the window.
    • ON_TIME

      public static final PaneInfo.Timing ON_TIME
      Pane was fired by a AfterWatermark.pastEndOfWindow() trigger because the input watermark progressed after the end of the window. However the output watermark has not yet progressed after the end of the window. Thus it is still possible to assign a timestamp to the element representing this pane which cannot be considered locally late by any following computation.
    • LATE

      public static final PaneInfo.Timing LATE
      Pane was fired after the output watermark had progressed past the end of the window.
    • UNKNOWN

      public static final PaneInfo.Timing UNKNOWN
      This element was not produced in a triggered pane and its relation to input and output watermarks is unknown.
  • Method Details

    • values

      public static PaneInfo.Timing[] values()
      Returns an array containing the constants of this enum class, in the order they are declared.
      Returns:
      an array containing the constants of this enum class, in the order they are declared
    • valueOf

      public static PaneInfo.Timing valueOf(String name)
      Returns the enum constant of this class with the specified name. The string must match exactly an identifier used to declare an enum constant in this class. (Extraneous whitespace characters are not permitted.)
      Parameters:
      name - the name of the enum constant to be returned.
      Returns:
      the enum constant with the specified name
      Throws:
      IllegalArgumentException - if this enum class has no constant with the specified name
      NullPointerException - if the argument is null