K
- The key by which state is keyed.public interface KeyedStateBackend<K> extends KeyedStateFactory, PriorityQueueSetFactory, Disposable
Modifier and Type | Interface and Description |
---|---|
static interface |
KeyedStateBackend.KeySelectionListener<K>
Listener is given a callback when
setCurrentKey(K) is called (key context changes). |
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. |
boolean |
deregisterKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
Stop calling listener registered in
registerKeySelectionListener(org.apache.flink.runtime.state.KeyedStateBackend.KeySelectionListener<K>) . |
void |
dispose()
Disposes the object and releases all resources.
|
K |
getCurrentKey() |
<N> java.util.stream.Stream<K> |
getKeys(String state,
N namespace) |
<N> java.util.stream.Stream<Tuple2<K,N>> |
getKeysAndNamespaces(String state) |
TypeSerializer<K> |
getKeySerializer() |
<N,S extends State,T> |
getOrCreateKeyedState(TypeSerializer<N> namespaceSerializer,
StateDescriptor<S,T> 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)
Creates or retrieves a partitioned state backed by this state backend.
|
default boolean |
isSafeToReuseKVState()
Whether it's safe to reuse key-values from the state-backend, e.g for the purpose of
optimization.
|
default boolean |
isStateImmutableInStateBackend(CheckpointType checkpointOptions)
Deprecated.
|
void |
registerKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
State backend will call
KeyedStateBackend.KeySelectionListener.keySelected(K) when key context is switched
if supported. |
void |
setCurrentKey(K newKey)
Sets the current key that is used for partitioned state.
|
createOrUpdateInternalState, createOrUpdateInternalState, createOrUpdateInternalState
create, create
void setCurrentKey(K newKey)
newKey
- The new current key.K getCurrentKey()
TypeSerializer<K> getKeySerializer()
<N,S extends State,T> void applyToAllKeys(N namespace, TypeSerializer<N> namespaceSerializer, StateDescriptor<S,T> stateDescriptor, KeyedStateFunction<K,S> function) throws Exception
KeyedStateFunction
to the state with the provided StateDescriptor
of all the currently active keys.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
<N> java.util.stream.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.<N> java.util.stream.Stream<Tuple2<K,N>> getKeysAndNamespaces(String state)
state
- State variable for which existing keys will be returned.<N,S extends State,T> S getOrCreateKeyedState(TypeSerializer<N> namespaceSerializer, StateDescriptor<S,T> stateDescriptor) throws Exception
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.<N,S extends State> S getPartitionedState(N namespace, TypeSerializer<N> namespaceSerializer, StateDescriptor<S,?> stateDescriptor) throws Exception
TODO: NOTE: This method does a lot of work caching / retrieving states just to update the namespace. This method should be removed for the sake of namespaces being lazily fetched from the keyed state backend, or being set on the state directly.
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.void dispose()
Disposable
dispose
in interface Disposable
void registerKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
KeyedStateBackend.KeySelectionListener.keySelected(K)
when key context is switched
if supported.boolean deregisterKeySelectionListener(KeyedStateBackend.KeySelectionListener<K> listener)
registerKeySelectionListener(org.apache.flink.runtime.state.KeyedStateBackend.KeySelectionListener<K>)
.@Deprecated default boolean isStateImmutableInStateBackend(CheckpointType checkpointOptions)
default boolean isSafeToReuseKVState()
NOTE: this method should not be used to check for InternalPriorityQueue
, as the
priority queue could be stored on different locations, e.g RocksDB state-backend could store
that on JVM heap if configuring HEAP as the time-service factory.
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.