Interface InternalTimeServiceManager<K>
-
- Type Parameters:
K- The type of keys used for the timers and the registry.
- All Known Implementing Classes:
BatchExecutionInternalTimeServiceManager,InternalTimeServiceManagerImpl
@Internal public interface InternalTimeServiceManager<K>An entity keeping all the time-related services.NOTE: These services are only available to keyed operators.
-
-
Nested Class Summary
Nested Classes Modifier and Type Interface Description static interfaceInternalTimeServiceManager.ProviderA provider pattern for creating an instance of aInternalTimeServiceManager.
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description voidadvanceWatermark(Watermark watermark)Advances the Watermark of all managedtimer services, potentially firing event time timers.<N> InternalTimerService<N>getInternalTimerService(String name, org.apache.flink.api.common.typeutils.TypeSerializer<K> keySerializer, org.apache.flink.api.common.typeutils.TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable)Creates anInternalTimerServicefor handling a group of timers identified by the givenname.voidsnapshotToRawKeyedState(org.apache.flink.runtime.state.KeyedStateCheckpointOutputStream stateCheckpointOutputStream, String operatorName)Snapshots the timers to raw keyed state.
-
-
-
Method Detail
-
getInternalTimerService
<N> InternalTimerService<N> getInternalTimerService(String name, org.apache.flink.api.common.typeutils.TypeSerializer<K> keySerializer, org.apache.flink.api.common.typeutils.TypeSerializer<N> namespaceSerializer, Triggerable<K,N> triggerable)
Creates anInternalTimerServicefor handling a group of timers identified by the givenname. The timers are scoped to a key and namespace.When a timer fires the given
Triggerablewill be invoked.
-
advanceWatermark
void advanceWatermark(Watermark watermark) throws Exception
Advances the Watermark of all managedtimer services, potentially firing event time timers.- Throws:
Exception
-
snapshotToRawKeyedState
void snapshotToRawKeyedState(org.apache.flink.runtime.state.KeyedStateCheckpointOutputStream stateCheckpointOutputStream, String operatorName) throws ExceptionSnapshots the timers to raw keyed state.TODO: This can be removed once heap-based timers are integrated with RocksDB incremental snapshots.
- Throws:
Exception
-
-