K
- Type of the key by which state is keyed.public abstract class AbstractKeyedStateBackend<K> extends Object implements CheckpointableKeyedStateBackend<K>, InternalCheckpointListener, TestableKeyedStateBackend<K>, InternalKeyContext<K>
Snapshotable.snapshot(long, long, CheckpointStreamFactory, CheckpointOptions)
.Modifier and Type | Class and Description |
---|---|
static interface |
AbstractKeyedStateBackend.PartitionStateFactory |
KeyedStateBackend.KeySelectionListener<K>
Modifier and Type | Field and Description |
---|---|
protected CloseableRegistry |
cancelStreamRegistry
Registry for all opened streams, so they can be closed if the task using this backend is
closed.
|
protected InternalKeyContext<K> |
keyContext
The key context for this backend.
|
protected StreamCompressionDecorator |
keyGroupCompressionDecorator
Decorates the input and output streams to write key-groups compressed.
|
protected KeyGroupRange |
keyGroupRange
Range of key-groups for which this backend is responsible.
|
protected TypeSerializer<K> |
keySerializer
The key serializer.
|
protected TaskKvStateRegistry |
kvStateRegistry
KvStateRegistry helper for this task.
|
protected LatencyTrackingStateConfig |
latencyTrackingStateConfig |
protected int |
numberOfKeyGroups
The number of key-groups aka max parallelism.
|
protected TtlTimeProvider |
ttlTimeProvider |
protected ClassLoader |
userCodeClassLoader |
Modifier | Constructor and Description |
---|---|
protected |
AbstractKeyedStateBackend(AbstractKeyedStateBackend<K> abstractKeyedStateBackend) |
|
AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry,
TypeSerializer<K> keySerializer,
ClassLoader userCodeClassLoader,
ExecutionConfig executionConfig,
TtlTimeProvider ttlTimeProvider,
LatencyTrackingStateConfig latencyTrackingStateConfig,
CloseableRegistry cancelStreamRegistry,
InternalKeyContext<K> keyContext) |
|
AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry,
TypeSerializer<K> keySerializer,
ClassLoader userCodeClassLoader,
ExecutionConfig executionConfig,
TtlTimeProvider ttlTimeProvider,
LatencyTrackingStateConfig latencyTrackingStateConfig,
CloseableRegistry cancelStreamRegistry,
StreamCompressionDecorator keyGroupCompressionDecorator,
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)
Applies the provided
KeyedStateFunction to the state with the provided StateDescriptor of all the currently active keys. |
<N,S extends State,T> |
applyToAllKeys(N namespace,
TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,T> stateDescriptor,
KeyedStateFunction<K,S> function,
AbstractKeyedStateBackend.PartitionStateFactory partitionStateFactory) |
void |
close() |
boolean |
deregisterKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
Stop calling listener registered in
KeyedStateBackend.registerKeySelectionListener(org.apache.flink.runtime.state.KeyedStateBackend.KeySelectionListener<K>) . |
void |
dispose()
Closes the state backend, releasing all internal resources, but does not delete any
persistent checkpoint data.
|
K |
getCurrentKey()
Used by states to access the current key.
|
int |
getCurrentKeyGroupIndex()
Returns the key-group to which the current key belongs.
|
InternalKeyContext<K> |
getKeyContext() |
StreamCompressionDecorator |
getKeyGroupCompressionDecorator() |
KeyGroupRange |
getKeyGroupRange()
Returns the key groups which this state backend is responsible for.
|
TypeSerializer<K> |
getKeySerializer() |
LatencyTrackingStateConfig |
getLatencyTrackingStateConfig() |
int |
getNumberOfKeyGroups()
Returns the number of key-groups aka max parallelism.
|
<N,S extends State,V> |
getOrCreateKeyedState(TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,V> stateDescriptor)
Creates or retrieves a keyed state backed by this state backend.
|
<N,S extends State> |
getPartitionedState(N namespace,
TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,?> stateDescriptor)
TODO: NOTE: This method does a lot of work caching / retrieving states just to update the
namespace.
|
void |
notifyCheckpointSubsumed(long checkpointId)
This method is called as a notification once a distributed checkpoint has been subsumed.
|
int |
numKeyValueStatesByName() |
void |
publishQueryableStateIfEnabled(StateDescriptor<?,?> stateDescriptor,
InternalKvState<?,?,?> kvState) |
void |
registerKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
State backend will call
KeyedStateBackend.KeySelectionListener.keySelected(K) when key context is switched
if supported. |
boolean |
requiresLegacySynchronousTimerSnapshots(SnapshotType checkpointType) |
void |
setCurrentKey(K newKey)
Sets the current key that is used for partitioned state.
|
void |
setCurrentKeyGroupIndex(int currentKeyGroupIndex)
Set current key group index of the context.
|
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
savepoint
snapshot
notifyCheckpointAborted, notifyCheckpointComplete
getDelegatedKeyedStateBackend, numKeyValueStateEntries
getKeys, getKeysAndNamespaces, isSafeToReuseKVState, isStateImmutableInStateBackend
createOrUpdateInternalState, createOrUpdateInternalState, createOrUpdateInternalState
create, create
protected final TypeSerializer<K> keySerializer
protected final int numberOfKeyGroups
protected final KeyGroupRange keyGroupRange
protected final TaskKvStateRegistry kvStateRegistry
protected CloseableRegistry cancelStreamRegistry
protected final ClassLoader userCodeClassLoader
protected final TtlTimeProvider ttlTimeProvider
protected final LatencyTrackingStateConfig latencyTrackingStateConfig
protected final StreamCompressionDecorator keyGroupCompressionDecorator
protected final InternalKeyContext<K> keyContext
public AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry, TypeSerializer<K> keySerializer, ClassLoader userCodeClassLoader, ExecutionConfig executionConfig, TtlTimeProvider ttlTimeProvider, LatencyTrackingStateConfig latencyTrackingStateConfig, CloseableRegistry cancelStreamRegistry, InternalKeyContext<K> keyContext)
public AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry, TypeSerializer<K> keySerializer, ClassLoader userCodeClassLoader, ExecutionConfig executionConfig, TtlTimeProvider ttlTimeProvider, LatencyTrackingStateConfig latencyTrackingStateConfig, CloseableRegistry cancelStreamRegistry, StreamCompressionDecorator keyGroupCompressionDecorator, InternalKeyContext<K> keyContext)
protected AbstractKeyedStateBackend(AbstractKeyedStateBackend<K> abstractKeyedStateBackend)
public void notifyCheckpointSubsumed(long checkpointId) throws Exception
InternalCheckpointListener
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.
notifyCheckpointSubsumed
in interface InternalCheckpointListener
checkpointId
- The ID of the checkpoint that has been subsumed.Exception
- This method can propagate exceptions, which leads to a failure/recovery for
the task or job.public void dispose()
dispose
in interface KeyedStateBackend<K>
dispose
in interface Disposable
public void setCurrentKey(K newKey)
KeyedStateBackend
setCurrentKey
in interface InternalKeyContext<K>
setCurrentKey
in interface KeyedStateBackend<K>
newKey
- The new current key.KeyedStateBackend
public void registerKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
KeyedStateBackend
KeyedStateBackend.KeySelectionListener.keySelected(K)
when key context is switched
if supported.registerKeySelectionListener
in interface KeyedStateBackend<K>
public boolean deregisterKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
KeyedStateBackend
KeyedStateBackend.registerKeySelectionListener(org.apache.flink.runtime.state.KeyedStateBackend.KeySelectionListener<K>)
.deregisterKeySelectionListener
in interface KeyedStateBackend<K>
public TypeSerializer<K> getKeySerializer()
getKeySerializer
in interface KeyedStateBackend<K>
KeyedStateBackend
public K getCurrentKey()
InternalKeyContext
getCurrentKey
in interface InternalKeyContext<K>
getCurrentKey
in interface KeyedStateBackend<K>
KeyedStateBackend
public int getCurrentKeyGroupIndex()
InternalKeyContext
getCurrentKeyGroupIndex
in interface InternalKeyContext<K>
KeyedStateBackend
public int getNumberOfKeyGroups()
InternalKeyContext
getNumberOfKeyGroups
in interface InternalKeyContext<K>
KeyedStateBackend
public KeyGroupRange getKeyGroupRange()
CheckpointableKeyedStateBackend
getKeyGroupRange
in interface CheckpointableKeyedStateBackend<K>
getKeyGroupRange
in interface InternalKeyContext<K>
KeyedStateBackend
public <N,S extends State,T> void applyToAllKeys(N namespace, TypeSerializer<N> namespaceSerializer, StateDescriptor<S,T> stateDescriptor, KeyedStateFunction<K,S> function) throws Exception
KeyedStateBackend
KeyedStateFunction
to the state with the provided StateDescriptor
of all the currently active keys.applyToAllKeys
in interface KeyedStateBackend<K>
N
- The type of the namespace.S
- The type of the state.namespace
- the namespace of the state.namespaceSerializer
- the serializer for the namespace.stateDescriptor
- the descriptor of the state to which the function is going to be
applied.function
- the function to be applied to the keyed state.Exception
KeyedStateBackend
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
Exception
public <N,S extends State,V> S getOrCreateKeyedState(TypeSerializer<N> namespaceSerializer, StateDescriptor<S,V> stateDescriptor) throws Exception
KeyedStateBackend
getOrCreateKeyedState
in interface KeyedStateBackend<K>
N
- The type of the namespace.S
- The type of the state.namespaceSerializer
- The serializer used for the namespace type of the statestateDescriptor
- The identifier for the state. This contains name and can create a
default state value.Exception
- Exceptions may occur during initialization of the state and should be
forwarded.KeyedStateBackend
public void publishQueryableStateIfEnabled(StateDescriptor<?,?> stateDescriptor, InternalKvState<?,?,?> kvState)
public <N,S extends State> S getPartitionedState(N namespace, TypeSerializer<N> namespaceSerializer, StateDescriptor<S,?> stateDescriptor) throws Exception
getPartitionedState
in interface KeyedStateBackend<K>
N
- The type of the namespace.S
- The type of the state.stateDescriptor
- The identifier for the state. This contains name and can create a
default state value.Exception
- Exceptions may occur during initialization of the state and should be
forwarded.KeyedStateBackend
public void close() throws IOException
close
in interface Closeable
close
in interface AutoCloseable
IOException
public LatencyTrackingStateConfig getLatencyTrackingStateConfig()
@VisibleForTesting public StreamCompressionDecorator getKeyGroupCompressionDecorator()
@VisibleForTesting public int numKeyValueStatesByName()
public boolean requiresLegacySynchronousTimerSnapshots(SnapshotType checkpointType)
public InternalKeyContext<K> getKeyContext()
public void setCurrentKeyGroupIndex(int currentKeyGroupIndex)
InternalKeyContext
setCurrentKeyGroupIndex
in interface InternalKeyContext<K>
currentKeyGroupIndex
- the current key group index to set to.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.