Class BatchExecutionInternalTimeServiceManager<K>
- java.lang.Object
-
- org.apache.flink.streaming.api.operators.sorted.state.BatchExecutionInternalTimeServiceManager<K>
-
- All Implemented Interfaces:
KeyedStateBackend.KeySelectionListener<K>
,InternalTimeServiceManager<K>
public class BatchExecutionInternalTimeServiceManager<K> extends Object implements InternalTimeServiceManager<K>, KeyedStateBackend.KeySelectionListener<K>
An implementation of aInternalTimeServiceManager
that manages timers with a single active key at a time. Can be used in a BATCH execution mode.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.streaming.api.operators.InternalTimeServiceManager
InternalTimeServiceManager.Provider, InternalTimeServiceManager.ShouldStopAdvancingFn
-
-
Constructor Summary
Constructors Constructor Description BatchExecutionInternalTimeServiceManager(ProcessingTimeService processingTimeService)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description void
advanceWatermark(Watermark watermark)
Advances the Watermark of all managedtimer services
, potentially firing event time timers.static <K> InternalTimeServiceManager<K>
create(TaskIOMetricGroup taskIOMetricGroup, PriorityQueueSetFactory factory, KeyGroupRange keyGroupRange, ClassLoader userClassloader, KeyContext keyContext, ProcessingTimeService processingTimeService, Iterable<KeyGroupStatePartitionStreamProvider> rawKeyedStates, StreamTaskCancellationContext cancellationContext)
<N> InternalTimerService<N>
getAsyncInternalTimerService(String name, TypeSerializer<K> keySerializer, TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable, AsyncExecutionController<K> asyncExecutionController)
Creates anInternalTimerServiceAsyncImpl
for handling a group of timers identified by the givenname
.<N> InternalTimerService<N>
getInternalTimerService(String name, TypeSerializer<K> keySerializer, TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable)
Creates anInternalTimerService
for handling a group of timers identified by the givenname
.void
keySelected(K newKey)
Callback when key context is switched.void
snapshotToRawKeyedState(KeyedStateCheckpointOutputStream context, String operatorName)
Snapshots the timers to raw keyed state.boolean
tryAdvanceWatermark(Watermark watermark, InternalTimeServiceManager.ShouldStopAdvancingFn shouldStopAdvancingFn)
Try toInternalTimeServiceManager.advanceWatermark(Watermark)
, but ifInternalTimeServiceManager.ShouldStopAdvancingFn
returnstrue
, stop the advancement and return as soon as possible.
-
-
-
Constructor Detail
-
BatchExecutionInternalTimeServiceManager
public BatchExecutionInternalTimeServiceManager(ProcessingTimeService processingTimeService)
-
-
Method Detail
-
getInternalTimerService
public <N> InternalTimerService<N> getInternalTimerService(String name, TypeSerializer<K> keySerializer, TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable)
Description copied from interface:InternalTimeServiceManager
Creates anInternalTimerService
for handling a group of timers identified by the givenname
. The timers are scoped to a key and namespace.When a timer fires the given
Triggerable
will be invoked.- Specified by:
getInternalTimerService
in interfaceInternalTimeServiceManager<K>
-
getAsyncInternalTimerService
public <N> InternalTimerService<N> getAsyncInternalTimerService(String name, TypeSerializer<K> keySerializer, TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable, AsyncExecutionController<K> asyncExecutionController)
Description copied from interface:InternalTimeServiceManager
Creates anInternalTimerServiceAsyncImpl
for handling a group of timers identified by the givenname
. The timers are scoped to a key and namespace. Mainly used by async operators.Some essential order preservation will be added when the given
Triggerable
is invoked.- Specified by:
getAsyncInternalTimerService
in interfaceInternalTimeServiceManager<K>
-
advanceWatermark
public void advanceWatermark(Watermark watermark)
Description copied from interface:InternalTimeServiceManager
Advances the Watermark of all managedtimer services
, potentially firing event time timers.- Specified by:
advanceWatermark
in interfaceInternalTimeServiceManager<K>
-
tryAdvanceWatermark
public boolean tryAdvanceWatermark(Watermark watermark, InternalTimeServiceManager.ShouldStopAdvancingFn shouldStopAdvancingFn)
Description copied from interface:InternalTimeServiceManager
Try toInternalTimeServiceManager.advanceWatermark(Watermark)
, but ifInternalTimeServiceManager.ShouldStopAdvancingFn
returnstrue
, stop the advancement and return as soon as possible.- Specified by:
tryAdvanceWatermark
in interfaceInternalTimeServiceManager<K>
- Returns:
- true if
Watermark
has been fully processed, false otherwise.
-
snapshotToRawKeyedState
public void snapshotToRawKeyedState(KeyedStateCheckpointOutputStream context, String operatorName) throws Exception
Description copied from interface:InternalTimeServiceManager
Snapshots the timers to raw keyed state.TODO: This can be removed once heap-based timers are integrated with RocksDB incremental snapshots.
- Specified by:
snapshotToRawKeyedState
in interfaceInternalTimeServiceManager<K>
- Throws:
Exception
-
create
public static <K> InternalTimeServiceManager<K> create(TaskIOMetricGroup taskIOMetricGroup, PriorityQueueSetFactory factory, KeyGroupRange keyGroupRange, ClassLoader userClassloader, KeyContext keyContext, ProcessingTimeService processingTimeService, Iterable<KeyGroupStatePartitionStreamProvider> rawKeyedStates, StreamTaskCancellationContext cancellationContext)
-
keySelected
public void keySelected(K newKey)
Description copied from interface:KeyedStateBackend.KeySelectionListener
Callback when key context is switched.- Specified by:
keySelected
in interfaceKeyedStateBackend.KeySelectionListener<K>
-
-