@Experimental(value=OUTPUT_TIME) public enum TimestampCombiner extends java.lang.Enum<TimestampCombiner>
In particular, these govern the timestamp on the output of a grouping transform such as GroupByKey
or Combine
.
Enum Constant and Description |
---|
EARLIEST
The policy of taking at the earliest of a set of timestamps.
|
END_OF_WINDOW
The policy of using the end of the window, regardless of input timestamps.
|
LATEST
The policy of taking the latest of a set of timestamps.
|
Modifier and Type | Method and Description |
---|---|
Instant |
assign(BoundedWindow intoWindow,
Instant timestamp)
Shorthand for
merge(org.apache.beam.sdk.transforms.windowing.BoundedWindow, java.lang.Iterable<? extends org.joda.time.Instant>) with just one element, to place it into the context of a window. |
Instant |
combine(Instant... timestamps)
Varargs variant of
combine(java.lang.Iterable<? extends org.joda.time.Instant>) . |
abstract Instant |
combine(java.lang.Iterable<? extends Instant> timestamps)
Combines the given times, which must be from the same window and must have been passed through
merge(org.apache.beam.sdk.transforms.windowing.BoundedWindow, java.lang.Iterable<? extends org.joda.time.Instant>) . |
abstract boolean |
dependsOnlyOnEarliestTimestamp()
Returns
true if the result of combination of many output timestamps actually depends
only on the earliest. |
abstract boolean |
dependsOnlyOnWindow()
Returns
true if the result does not depend on what outputs were combined but only the
window they are in. |
Instant |
merge(BoundedWindow intoWindow,
Instant... timestamps)
|
abstract Instant |
merge(BoundedWindow intoWindow,
java.lang.Iterable<? extends Instant> mergingTimestamps)
Merges the given timestamps, which may have originated in separate windows, into the context of
the result window.
|
static TimestampCombiner |
valueOf(java.lang.String name)
Returns the enum constant of this type with the specified name.
|
static TimestampCombiner[] |
values()
Returns an array containing the constants of this enum type, in
the order they are declared.
|
public static final TimestampCombiner EARLIEST
When used in windowed aggregations, the timestamps of non-late inputs will be combined after
they are shifted by the WindowFn
(to allow downstream watermark progress).
If data arrives late, it has no effect on the output timestamp.
public static final TimestampCombiner LATEST
When used in windowed aggregations, the timestamps of non-late inputs will be combined after
they are shifted by the WindowFn
(to allow downstream watermark progress).
If data arrives late, it has no effect on the output timestamp.
public static final TimestampCombiner END_OF_WINDOW
When used in windowed aggregations, the timestamps of non-late inputs will be combined after
they are shifted by the WindowFn
(to allow downstream watermark progress).
If data arrives late, it has no effect on the output timestamp.
public static TimestampCombiner[] values()
for (TimestampCombiner c : TimestampCombiner.values()) System.out.println(c);
public static TimestampCombiner valueOf(java.lang.String name)
name
- the name of the enum constant to be returned.java.lang.IllegalArgumentException
- if this enum type has no constant with the specified namejava.lang.NullPointerException
- if the argument is nullpublic abstract Instant combine(java.lang.Iterable<? extends Instant> timestamps)
merge(org.apache.beam.sdk.transforms.windowing.BoundedWindow, java.lang.Iterable<? extends org.joda.time.Instant>)
.
combine
must be commutative: combine(a, b).equals(combine(b, a))
.
combine
must be associative: combine(a, combine(b,
c)).equals(combine(combine(a, b), c))
.
public abstract Instant merge(BoundedWindow intoWindow, java.lang.Iterable<? extends Instant> mergingTimestamps)
public final Instant assign(BoundedWindow intoWindow, Instant timestamp)
merge(org.apache.beam.sdk.transforms.windowing.BoundedWindow, java.lang.Iterable<? extends org.joda.time.Instant>)
with just one element, to place it into the context of a window.
For example, the END_OF_WINDOW
policy moves the timestamp to the end of the window.
public final Instant combine(Instant... timestamps)
combine(java.lang.Iterable<? extends org.joda.time.Instant>)
.public final Instant merge(BoundedWindow intoWindow, Instant... timestamps)
public abstract boolean dependsOnlyOnEarliestTimestamp()
true
if the result of combination of many output timestamps actually depends
only on the earliest.
This may allow optimizations when it is very efficient to retrieve the earliest timestamp to be combined.
public abstract boolean dependsOnlyOnWindow()
true
if the result does not depend on what outputs were combined but only the
window they are in. The canonical example is if all timestamps are sure to be the end of the
window.
This may allow optimizations, since it is typically very efficient to retrieve the window and combining output timestamps is not necessary.