public abstract class BoundedOutOfOrdernessTimestampExtractor<T> extends Object implements AssignerWithPeriodicWatermarks<T>
AssignerWithPeriodicWatermarks
used to emit Watermarks that lag behind the
element with the maximum timestamp (in event time) seen so far by a fixed amount of time,
t_late
. This can help reduce the number of elements that are ignored due to lateness when
computing the final result for a given window, in the case where we know that elements arrive no
later than t_late
units of time after the watermark that signals that the system
event-time has advanced past their (event-time) timestamp.NO_TIMESTAMP
Constructor and Description |
---|
BoundedOutOfOrdernessTimestampExtractor(Time maxOutOfOrderness) |
Modifier and Type | Method and Description |
---|---|
abstract long |
extractTimestamp(T element)
Extracts the timestamp from the given element.
|
long |
extractTimestamp(T element,
long previousElementTimestamp)
Assigns a timestamp to an element, in milliseconds since the Epoch.
|
Watermark |
getCurrentWatermark()
Returns the current watermark.
|
long |
getMaxOutOfOrdernessInMillis() |
public BoundedOutOfOrdernessTimestampExtractor(Time maxOutOfOrderness)
public long getMaxOutOfOrdernessInMillis()
public abstract long extractTimestamp(T element)
element
- The element that the timestamp is extracted from.public final Watermark getCurrentWatermark()
AssignerWithPeriodicWatermarks
null
to indicate that no new Watermark
is available.
The returned watermark will be emitted only if it is non-null and its timestamp is larger than that of the previously emitted watermark (to preserve the contract of ascending watermarks). If the current watermark is still identical to the previous one, no progress in event time has happened since the previous call to this method. If a null value is returned, or the timestamp of the returned watermark is smaller than that of the last emitted one, then no new watermark will be generated.
The interval in which this method is called and Watermarks are generated depends on ExecutionConfig.getAutoWatermarkInterval()
.
getCurrentWatermark
in interface AssignerWithPeriodicWatermarks<T>
Null
, if no watermark should be emitted, or the next watermark to emit.Watermark
,
ExecutionConfig.getAutoWatermarkInterval()
public final long extractTimestamp(T element, long previousElementTimestamp)
TimestampAssigner
The method is passed the previously assigned timestamp of the element. That previous
timestamp may have been assigned from a previous assigner, by ingestion time. If the element
did not carry a timestamp before, this value is Long.MIN_VALUE
.
extractTimestamp
in interface TimestampAssigner<T>
element
- The element that the timestamp will be assigned to.previousElementTimestamp
- The previous internal timestamp of the element, or a negative value,
if no timestamp has been assigned yet.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.