K
- Type of the key by which state is keyed.public abstract class AbstractKeyedStateBackend<K> extends Object implements KeyedStateBackend<K>, Snapshotable<KeyedStateHandle>, Closeable, CheckpointListener
Snapshotable.snapshot(long, long, CheckpointStreamFactory, CheckpointOptions)
.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 K |
currentKey
The currently active key.
|
protected StreamCompressionDecorator |
keyGroupCompressionDecorator
Decoratores 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
TypeSerializer for our key. |
protected HashMap<String,InternalKvState<?>> |
keyValueStatesByName
So that we can give out state when the user uses the same key.
|
protected TaskKvStateRegistry |
kvStateRegistry
KvStateRegistry helper for this task
|
protected int |
numberOfKeyGroups
The number of key-groups aka max parallelism
|
protected ClassLoader |
userCodeClassLoader |
Constructor and Description |
---|
AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry,
TypeSerializer<K> keySerializer,
ClassLoader userCodeClassLoader,
int numberOfKeyGroups,
KeyGroupRange keyGroupRange,
ExecutionConfig executionConfig) |
Modifier and Type | Method and Description |
---|---|
void |
close() |
protected abstract <N,T,ACC,R> |
createAggregatingState(TypeSerializer<N> namespaceSerializer,
AggregatingStateDescriptor<T,ACC,R> stateDesc)
Creates and returns a new
AggregatingState . |
protected abstract <N,T,ACC> InternalFoldingState<N,T,ACC> |
createFoldingState(TypeSerializer<N> namespaceSerializer,
FoldingStateDescriptor<T,ACC> stateDesc)
Deprecated.
will be removed in a future version
|
protected abstract <N,T> InternalListState<N,T> |
createListState(TypeSerializer<N> namespaceSerializer,
ListStateDescriptor<T> stateDesc)
Creates and returns a new
ListState . |
protected abstract <N,UK,UV> InternalMapState<N,UK,UV> |
createMapState(TypeSerializer<N> namespaceSerializer,
MapStateDescriptor<UK,UV> stateDesc)
Creates and returns a new
MapState . |
protected abstract <N,T> InternalReducingState<N,T> |
createReducingState(TypeSerializer<N> namespaceSerializer,
ReducingStateDescriptor<T> stateDesc)
Creates and returns a new
ReducingState . |
protected abstract <N,T> InternalValueState<N,T> |
createValueState(TypeSerializer<N> namespaceSerializer,
ValueStateDescriptor<T> stateDesc)
Creates and returns a new
ValueState . |
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.
|
StreamCompressionDecorator |
getKeyGroupCompressionDecorator() |
KeyGroupRange |
getKeyGroupRange()
Returns the key groups for this backend.
|
TypeSerializer<K> |
getKeySerializer()
TypeSerializer for the state backend key type. |
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.
|
abstract int |
numStateEntries()
Returns the total number of state entries across all keys/namespaces.
|
void |
setCurrentKey(K newKey)
Sets the current key that is used for partitioned state.
|
boolean |
supportsAsynchronousSnapshots() |
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
getKeys
restore, snapshot
notifyCheckpointComplete
protected final TypeSerializer<K> keySerializer
TypeSerializer
for our key.protected K currentKey
protected final HashMap<String,InternalKvState<?>> keyValueStatesByName
protected final int numberOfKeyGroups
protected final KeyGroupRange keyGroupRange
protected final TaskKvStateRegistry kvStateRegistry
protected CloseableRegistry cancelStreamRegistry
protected final ClassLoader userCodeClassLoader
protected final StreamCompressionDecorator keyGroupCompressionDecorator
public AbstractKeyedStateBackend(TaskKvStateRegistry kvStateRegistry, TypeSerializer<K> keySerializer, ClassLoader userCodeClassLoader, int numberOfKeyGroups, KeyGroupRange keyGroupRange, ExecutionConfig executionConfig)
public void dispose()
dispose
in interface KeyedStateBackend<K>
protected abstract <N,T> InternalValueState<N,T> createValueState(TypeSerializer<N> namespaceSerializer, ValueStateDescriptor<T> stateDesc) throws Exception
ValueState
.N
- The type of the namespace.T
- The type of the value that the ValueState
can store.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
protected abstract <N,T> InternalListState<N,T> createListState(TypeSerializer<N> namespaceSerializer, ListStateDescriptor<T> stateDesc) throws Exception
ListState
.N
- The type of the namespace.T
- The type of the values that the ListState
can store.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
protected abstract <N,T> InternalReducingState<N,T> createReducingState(TypeSerializer<N> namespaceSerializer, ReducingStateDescriptor<T> stateDesc) throws Exception
ReducingState
.N
- The type of the namespace.T
- The type of the values that the ListState
can store.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
protected abstract <N,T,ACC,R> InternalAggregatingState<N,T,R> createAggregatingState(TypeSerializer<N> namespaceSerializer, AggregatingStateDescriptor<T,ACC,R> stateDesc) throws Exception
AggregatingState
.N
- The type of the namespace.T
- The type of the values that the ListState
can store.namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
@Deprecated protected abstract <N,T,ACC> InternalFoldingState<N,T,ACC> createFoldingState(TypeSerializer<N> namespaceSerializer, FoldingStateDescriptor<T,ACC> stateDesc) throws Exception
FoldingState
.N
- The type of the namespace.T
- Type of the values folded into the stateACC
- Type of the value in the statenamespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
protected abstract <N,UK,UV> InternalMapState<N,UK,UV> createMapState(TypeSerializer<N> namespaceSerializer, MapStateDescriptor<UK,UV> stateDesc) throws Exception
MapState
.N
- The type of the namespace.UK
- Type of the keys in the stateUV
- Type of the values in the state *namespaceSerializer
- TypeSerializer for the state namespace.stateDesc
- The StateDescriptor
that contains the name of the state.Exception
public void setCurrentKey(K newKey)
KeyedStateBackend
setCurrentKey
in interface KeyedStateBackend<K>
newKey
- The new current key.KeyedStateBackend
public TypeSerializer<K> getKeySerializer()
InternalKeyContext
TypeSerializer
for the state backend key type.getKeySerializer
in interface InternalKeyContext<K>
KeyedStateBackend
public K getCurrentKey()
InternalKeyContext
getCurrentKey
in interface InternalKeyContext<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()
InternalKeyContext
getKeyGroupRange
in interface InternalKeyContext<K>
KeyedStateBackend
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 <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
@VisibleForTesting public boolean supportsAsynchronousSnapshots()
@VisibleForTesting public StreamCompressionDecorator getKeyGroupCompressionDecorator()
@VisibleForTesting public abstract int numStateEntries()
Copyright © 2014–2018 The Apache Software Foundation. All rights reserved.