Package org.apache.beam.sdk.transforms
Annotation Interface DoFn.GetInitialWatermarkEstimatorState
@Documented
@Retention(RUNTIME)
@Target(METHOD)
public static @interface DoFn.GetInitialWatermarkEstimatorState
Annotation for the method that maps an element and restriction to initial watermark estimator
state for a splittable
DoFn.
Signature: WatermarkEstimatorStateT getInitialWatermarkState(<arguments>);
This method must satisfy the following constraints:
- The return type
WatermarkEstimatorStateTdefines the watermark state type used within this splittable DoFn. The return type is allowed to be nullable. All other methods that use a@WatermarkEstimatorStateparameter must use the same type that is used here. It is suggested to use as narrow of a return type definition as possible (for example prefer to use a square type over a shape type as a square is a type of a shape). - If one of its arguments is tagged with the
DoFn.Elementannotation, then it will be passed the current element being processed; the argument must be of typeInputT. Note that automatic conversion ofRows andDoFn.FieldAccessparameters are currently unsupported. - If one of its arguments is tagged with the
DoFn.Restrictionannotation, then it will be passed the current restriction being processed; the argument must be of typeRestrictionT. - If one of its arguments is tagged with the
DoFn.Timestampannotation, then it will be passed the timestamp of the current element being processed; the argument must be of typeInstant. - If one of its arguments is a subtype of
BoundedWindow, then it will be passed the window of the current element. When applied byParDothe subtype ofBoundedWindowmust match the type of windows on the inputPCollection. If the window is not accessed a runner may perform additional optimizations. - If one of its arguments is of type
PaneInfo, then it will be passed information about the current triggering pane. - If one of the parameters is of type
PipelineOptions, then it will be passed the options for the current pipeline.