K
- The key by which state is keyed.public class HeapKeyedStateBackend<K> extends AbstractKeyedStateBackend<K>
AbstractKeyedStateBackend
that keeps state on the Java Heap and will serialize state to
streams provided by a CheckpointStreamFactory
upon checkpointing.AbstractKeyedStateBackend.PartitionStateFactory
KeyedStateBackend.KeySelectionListener<K>
cancelStreamRegistry, keyContext, keyGroupCompressionDecorator, keyGroupRange, keySerializer, kvStateRegistry, latencyTrackingStateConfig, numberOfKeyGroups, ttlTimeProvider, userCodeClassLoader
Constructor and Description |
---|
HeapKeyedStateBackend(TaskKvStateRegistry kvStateRegistry,
TypeSerializer<K> keySerializer,
ClassLoader userCodeClassLoader,
ExecutionConfig executionConfig,
TtlTimeProvider ttlTimeProvider,
LatencyTrackingStateConfig latencyTrackingStateConfig,
CloseableRegistry cancelStreamRegistry,
StreamCompressionDecorator keyGroupCompressionDecorator,
Map<String,StateTable<K,?,?>> registeredKVStates,
Map<String,HeapPriorityQueueSnapshotRestoreWrapper<?>> registeredPQStates,
LocalRecoveryConfig localRecoveryConfig,
HeapPriorityQueueSetFactory priorityQueueSetFactory,
org.apache.flink.runtime.state.heap.HeapSnapshotStrategy<K> checkpointStrategy,
SnapshotExecutionType snapshotExecutionType,
org.apache.flink.runtime.state.heap.StateTableFactory<K> stateTableFactory,
InternalKeyContext<K> keyContext) |
Modifier and Type | Method and Description |
---|---|
<N,S extends State,T> |
applyToAllKeys(N namespace,
TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,T> stateDescriptor,
KeyedStateFunction<K,S> function,
AbstractKeyedStateBackend.PartitionStateFactory partitionStateFactory) |
<T extends HeapPriorityQueueElement & PriorityComparable<? super T> & Keyed<?>> |
create(String stateName,
TypeSerializer<T> byteOrderedElementSerializer)
Creates a
KeyGroupedInternalPriorityQueue . |
<T extends HeapPriorityQueueElement & PriorityComparable<? super T> & Keyed<?>> |
create(String stateName,
TypeSerializer<T> byteOrderedElementSerializer,
boolean allowFutureMetadataUpdates)
Creates a
KeyGroupedInternalPriorityQueue . |
<N,SV,SEV,S extends State,IS extends S> |
createOrUpdateInternalState(TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,SV> stateDesc,
StateSnapshotTransformer.StateSnapshotTransformFactory<SEV> snapshotTransformFactory)
Creates or updates internal state and returns a new
InternalKvState . |
<N,SV,SEV,S extends State,IS extends S> |
createOrUpdateInternalState(TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,SV> stateDesc,
StateSnapshotTransformer.StateSnapshotTransformFactory<SEV> snapshotTransformFactory,
boolean allowFutureMetadataUpdates)
Creates or updates internal state and returns a new
InternalKvState . |
<N> Stream<K> |
getKeys(String state,
N namespace) |
<N> Stream<Tuple2<K,N>> |
getKeysAndNamespaces(String state) |
LocalRecoveryConfig |
getLocalRecoveryConfig() |
void |
notifyCheckpointAborted(long checkpointId)
This method is called as a notification once a distributed checkpoint has been aborted.
|
void |
notifyCheckpointComplete(long checkpointId)
Notifies the listener that the checkpoint with the given
checkpointId completed and
was committed. |
int |
numKeyValueStateEntries()
Returns the total number of state entries across all keys/namespaces.
|
int |
numKeyValueStateEntries(Object namespace)
Returns the total number of state entries across all keys for the given namespace.
|
SavepointResources<K> |
savepoint()
Returns a
SavepointResources that can be used by SavepointSnapshotStrategy to
write out a savepoint in the common/unified format. |
RunnableFuture<SnapshotResult<KeyedStateHandle>> |
snapshot(long checkpointId,
long timestamp,
CheckpointStreamFactory streamFactory,
CheckpointOptions checkpointOptions)
Operation that writes a snapshot into a stream that is provided by the given
CheckpointStreamFactory and returns a @RunnableFuture that gives a state handle to
the snapshot. |
String |
toString() |
applyToAllKeys, close, deregisterKeySelectionListener, dispose, getCurrentKey, getCurrentKeyGroupIndex, getKeyContext, getKeyGroupCompressionDecorator, getKeyGroupRange, getKeySerializer, getLatencyTrackingStateConfig, getNumberOfKeyGroups, getOrCreateKeyedState, getPartitionedState, notifyCheckpointSubsumed, numKeyValueStatesByName, publishQueryableStateIfEnabled, registerKeySelectionListener, requiresLegacySynchronousTimerSnapshots, setCurrentKey, setCurrentKeyGroupIndex
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
getDelegatedKeyedStateBackend
isSafeToReuseKVState, isStateImmutableInStateBackend
createOrUpdateInternalState
public HeapKeyedStateBackend(TaskKvStateRegistry kvStateRegistry, TypeSerializer<K> keySerializer, ClassLoader userCodeClassLoader, ExecutionConfig executionConfig, TtlTimeProvider ttlTimeProvider, LatencyTrackingStateConfig latencyTrackingStateConfig, CloseableRegistry cancelStreamRegistry, StreamCompressionDecorator keyGroupCompressionDecorator, Map<String,StateTable<K,?,?>> registeredKVStates, Map<String,HeapPriorityQueueSnapshotRestoreWrapper<?>> registeredPQStates, LocalRecoveryConfig localRecoveryConfig, HeapPriorityQueueSetFactory priorityQueueSetFactory, org.apache.flink.runtime.state.heap.HeapSnapshotStrategy<K> checkpointStrategy, SnapshotExecutionType snapshotExecutionType, org.apache.flink.runtime.state.heap.StateTableFactory<K> stateTableFactory, InternalKeyContext<K> keyContext)
@Nonnull public <T extends HeapPriorityQueueElement & PriorityComparable<? super T> & Keyed<?>> KeyGroupedInternalPriorityQueue<T> create(@Nonnull String stateName, @Nonnull TypeSerializer<T> byteOrderedElementSerializer)
PriorityQueueSetFactory
KeyGroupedInternalPriorityQueue
.T
- type of the stored elements.stateName
- unique name for associated with this queue.byteOrderedElementSerializer
- a serializer that with a format that is lexicographically
ordered in alignment with elementPriorityComparator.public <T extends HeapPriorityQueueElement & PriorityComparable<? super T> & Keyed<?>> KeyGroupedInternalPriorityQueue<T> create(@Nonnull String stateName, @Nonnull TypeSerializer<T> byteOrderedElementSerializer, boolean allowFutureMetadataUpdates)
PriorityQueueSetFactory
KeyGroupedInternalPriorityQueue
.T
- type of the stored elements.stateName
- unique name for associated with this queue.byteOrderedElementSerializer
- a serializer that with a format that is lexicographically
ordered in alignment with elementPriorityComparator.allowFutureMetadataUpdates
- whether allow metadata to update in the future or not.public <N> Stream<K> getKeys(String state, N namespace)
state
- State variable for which existing keys will be returned.namespace
- Namespace for which existing keys will be returned.public <N> Stream<Tuple2<K,N>> getKeysAndNamespaces(String state)
state
- State variable for which existing keys will be returned.@Nonnull public <N,SV,SEV,S extends State,IS extends S> IS createOrUpdateInternalState(@Nonnull TypeSerializer<N> namespaceSerializer, @Nonnull StateDescriptor<S,SV> stateDesc, @Nonnull StateSnapshotTransformer.StateSnapshotTransformFactory<SEV> snapshotTransformFactory) throws Exception
KeyedStateFactory
InternalKvState
.N
- The type of the namespace.SV
- The type of the stored state value.SEV
- The type of the stored state value or entry for collection types (list or map).S
- The type of the public API state.IS
- The type of internal state.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.snapshotTransformFactory
- factory of state snapshot transformer.Exception
@Nonnull public <N,SV,SEV,S extends State,IS extends S> IS createOrUpdateInternalState(@Nonnull TypeSerializer<N> namespaceSerializer, @Nonnull StateDescriptor<S,SV> stateDesc, @Nonnull StateSnapshotTransformer.StateSnapshotTransformFactory<SEV> snapshotTransformFactory, boolean allowFutureMetadataUpdates) throws Exception
KeyedStateFactory
InternalKvState
.N
- The type of the namespace.SV
- The type of the stored state value.SEV
- The type of the stored state value or entry for collection types (list or map).S
- The type of the public API state.IS
- The type of internal state.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.snapshotTransformFactory
- factory of state snapshot transformer.allowFutureMetadataUpdates
- whether allow metadata to update in the future or not.Exception
@Nonnull public RunnableFuture<SnapshotResult<KeyedStateHandle>> snapshot(long checkpointId, long timestamp, @Nonnull CheckpointStreamFactory streamFactory, @Nonnull CheckpointOptions checkpointOptions) throws Exception
Snapshotable
CheckpointStreamFactory
and returns a @RunnableFuture
that gives a state handle to
the snapshot. It is up to the implementation if the operation is performed synchronous or
asynchronous. In the later case, the returned Runnable must be executed first before
obtaining the handle.checkpointId
- The ID of the checkpoint.timestamp
- The timestamp of the checkpoint.streamFactory
- The factory that we can use for writing our state to streams.checkpointOptions
- Options for how to perform this checkpoint.StateObject
.Exception
@Nonnull public SavepointResources<K> savepoint()
CheckpointableKeyedStateBackend
SavepointResources
that can be used by SavepointSnapshotStrategy
to
write out a savepoint in the common/unified format.public void notifyCheckpointComplete(long checkpointId)
CheckpointListener
checkpointId
completed and
was committed.
These notifications are "best effort", meaning they can sometimes be skipped. To behave
properly, implementers need to follow the "Checkpoint Subsuming Contract". Please see the
class-level JavaDocs
for details.
Please note that checkpoints may generally overlap, so you cannot assume that the notifyCheckpointComplete()
call is always for the latest prior checkpoint (or snapshot) that
was taken on the function/operator implementing this interface. It might be for a checkpoint
that was triggered earlier. Implementing the "Checkpoint Subsuming Contract" (see above)
properly handles this situation correctly as well.
Please note that throwing exceptions from this method will not cause the completed checkpoint to be revoked. Throwing exceptions will typically cause task/job failure and trigger recovery.
checkpointId
- The ID of the checkpoint that has been completed.public void notifyCheckpointAborted(long checkpointId)
CheckpointListener
Important: The fact that a checkpoint has been aborted does NOT mean that the data
and artifacts produced between the previous checkpoint and the aborted checkpoint are to be
discarded. The expected behavior is as if this checkpoint was never triggered in the first
place, and the next successful checkpoint simply covers a longer time span. See the
"Checkpoint Subsuming Contract" in the class-level JavaDocs
for
details.
These notifications are "best effort", meaning they can sometimes be skipped.
This method is very rarely necessary to implement. The "best effort" guarantee, together with the fact that this method should not result in discarding any data (per the "Checkpoint Subsuming Contract") means it is mainly useful for earlier cleanups of auxiliary resources. One example is to pro-actively clear a local per-checkpoint state cache upon checkpoint failure.
checkpointId
- The ID of the checkpoint that has been aborted.public <N,S extends State,T> void applyToAllKeys(N namespace, TypeSerializer<N> namespaceSerializer, StateDescriptor<S,T> stateDescriptor, KeyedStateFunction<K,S> function, AbstractKeyedStateBackend.PartitionStateFactory partitionStateFactory) throws Exception
applyToAllKeys
in class AbstractKeyedStateBackend<K>
Exception
@VisibleForTesting public int numKeyValueStateEntries()
@VisibleForTesting public int numKeyValueStateEntries(Object namespace)
@VisibleForTesting public LocalRecoveryConfig getLocalRecoveryConfig()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.