public abstract class DoFn.WindowedContext
extends java.lang.Object
DoFn
where the context is in some window.Constructor and Description |
---|
WindowedContext() |
Modifier and Type | Method and Description |
---|---|
abstract PipelineOptions |
getPipelineOptions()
|
abstract void |
output(OutputT output)
Adds the given element to the main output
PCollection . |
abstract <T> void |
output(TupleTag<T> tag,
T output)
Adds the given element to the output
PCollection with the given tag. |
abstract void |
outputWithTimestamp(OutputT output,
Instant timestamp)
Adds the given element to the main output
PCollection , with the given timestamp. |
abstract <T> void |
outputWithTimestamp(TupleTag<T> tag,
T output,
Instant timestamp)
Adds the given element to the specified output
PCollection , with the given timestamp. |
@Pure public abstract PipelineOptions getPipelineOptions()
public abstract void output(OutputT output)
PCollection
.
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
or DoFn.FinishBundle
, this will attempt to use the
WindowFn
of the input PCollection
to
determine what windows the element should be in, throwing an exception if the WindowFn
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 from DoFn.StartBundle
or DoFn.FinishBundle
methods.
public abstract void outputWithTimestamp(OutputT output, Instant timestamp)
PCollection
, 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 minus DoFn.getAllowedTimestampSkew()
. The output element will be in
the same windows as the input element.
If invoked from DoFn.StartBundle
or DoFn.FinishBundle
, this will attempt to use the
WindowFn
of the input PCollection
to
determine what windows the element should be in, throwing an exception if the WindowFn
attempts to access any information about the input element except for the
timestamp.
Note: A splittable DoFn
is not allowed to output from DoFn.StartBundle
or DoFn.FinishBundle
methods.
public abstract <T> void output(TupleTag<T> tag, T output)
PCollection
with the given tag.
Once passed to output
the element should not be modified in any way.
The caller of ParDo
uses ParDo.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
or DoFn.FinishBundle
, this will attempt to use the
WindowFn
of the input PCollection
to
determine what windows the element should be in, throwing an exception if the WindowFn
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 from DoFn.StartBundle
or DoFn.FinishBundle
methods.
public abstract <T> void outputWithTimestamp(TupleTag<T> tag, T output, Instant timestamp)
PCollection
, 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 minus DoFn.getAllowedTimestampSkew()
. The output element will be in
the same windows as the input element.
If invoked from DoFn.StartBundle
or DoFn.FinishBundle
, this will attempt to use the
WindowFn
of the input PCollection
to
determine what windows the element should be in, throwing an exception if the WindowFn
attempts to access any information about the input element except for the
timestamp.
Note: A splittable DoFn
is not allowed to output from DoFn.StartBundle
or DoFn.FinishBundle
methods.