Class DoFn.WindowedContext
- Direct Known Subclasses:
DoFn.OnTimerContext
,DoFn.OnWindowExpirationContext
,DoFn.ProcessContext
DoFn
where the context is in some window.-
Constructor Summary
Constructors -
Method Summary
Modifier and TypeMethodDescriptionabstract PipelineOptions
abstract <T> void
Adds the given element to the outputPCollection
with the given tag.abstract void
Adds the given element to the main outputPCollection
.abstract <T> void
outputWindowedValue
(TupleTag<T> tag, T output, Instant timestamp, Collection<? extends BoundedWindow> windows, PaneInfo paneInfo) Adds the given element to the main outputPCollection
, with the given windowing metadata.abstract void
outputWindowedValue
(OutputT output, Instant timestamp, Collection<? extends BoundedWindow> windows, PaneInfo paneInfo) Adds the given element to the main outputPCollection
, with the given windowing metadata.abstract <T> void
outputWithTimestamp
(TupleTag<T> tag, T output, Instant timestamp) Adds the given element to the specified outputPCollection
, with the given timestamp.abstract void
outputWithTimestamp
(OutputT output, Instant timestamp) Adds the given element to the main outputPCollection
, with the given timestamp.
-
Constructor Details
-
WindowedContext
public WindowedContext()
-
-
Method Details
-
getPipelineOptions
-
output
Adds the given element to the main outputPCollection
.Once passed to
output
the element should not be modified in any way.If invoked from
DoFn.ProcessElement
, the output element will have the same timestamp and be in the same windows as the input element passed to the method annotated with@ProcessElement
.If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element. The output element will have a timestamp of negative infinity.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods. -
outputWithTimestamp
Adds the given element to the main outputPCollection
, with the given timestamp.Once passed to
outputWithTimestamp
the element should not be modified in any way.If invoked from
DoFn.ProcessElement
), the timestamp must not be older than the input element's timestamp minusDoFn.getAllowedTimestampSkew()
. The output element will be in the same windows as the input element.If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element except for the timestamp.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods. -
outputWindowedValue
public abstract void outputWindowedValue(OutputT output, Instant timestamp, Collection<? extends BoundedWindow> windows, PaneInfo paneInfo) Adds the given element to the main outputPCollection
, with the given windowing metadata.Once passed to
outputWindowedValue
the element should not be modified in any way.If invoked from
DoFn.ProcessElement
), the timestamp must not be older than the input element's timestamp minusDoFn.getAllowedTimestampSkew()
. The output element will be in the same windows as the input element.If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element except for the timestamp.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods. -
output
Adds the given element to the outputPCollection
with the given tag.Once passed to
output
the element should not be modified in any way.The caller of
ParDo
usesParDo.SingleOutput.withOutputTags(org.apache.beam.sdk.values.TupleTag<OutputT>, org.apache.beam.sdk.values.TupleTagList)
to specify the tags of outputs that it consumes. Non-consumed outputs, e.g., outputs for monitoring purposes only, don't necessarily need to be specified.The output element will have the same timestamp and be in the same windows as the input element passed to
DoFn.ProcessElement
).If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element. The output element will have a timestamp of negative infinity.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods.- See Also:
-
outputWithTimestamp
Adds the given element to the specified outputPCollection
, with the given timestamp.Once passed to
outputWithTimestamp
the element should not be modified in any way.If invoked from
DoFn.ProcessElement
), the timestamp must not be older than the input element's timestamp minusDoFn.getAllowedTimestampSkew()
. The output element will be in the same windows as the input element.If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element except for the timestamp.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods.- See Also:
-
outputWindowedValue
public abstract <T> void outputWindowedValue(TupleTag<T> tag, T output, Instant timestamp, Collection<? extends BoundedWindow> windows, PaneInfo paneInfo) Adds the given element to the main outputPCollection
, with the given windowing metadata.Once passed to
outputWindowedValue
the element should not be modified in any way.If invoked from
DoFn.ProcessElement
), the timestamp must not be older than the input element's timestamp minusDoFn.getAllowedTimestampSkew()
. The output element will be in the same windows as the input element.If invoked from
DoFn.StartBundle
orDoFn.FinishBundle
, this will attempt to use theWindowFn
of the inputPCollection
to determine what windows the element should be in, throwing an exception if theWindowFn
attempts to access any information about the input element except for the timestamp.Note: A splittable
DoFn
is not allowed to output fromDoFn.StartBundle
orDoFn.FinishBundle
methods.
-