Interface | Description |
---|---|
AbstractKeyedStateBackend.PartitionStateFactory | |
BackendWritableBroadcastState<K,V> |
An interface with methods related to the interplay between the
Broadcast
State and the OperatorStateBackend . |
CheckpointableKeyedStateBackend<K> |
Interface that combines both, the
KeyedStateBackend interface, which encapsulates methods
responsible for keyed state management and the Snapshotable which tells the system how to
snapshot the underlying state. |
CheckpointBoundKeyedStateHandle |
KeyedStateHandle that is bound to a specific checkpoint. |
CheckpointListener | Deprecated
This interface has been moved to
CheckpointListener . |
CheckpointStateToolset |
A toolset of operations that can be performed on a location embedded within the class.
|
CheckpointStorage |
CheckpointStorage defines how
StateBackend 's store their state for fault tolerance in
streaming applications. |
CheckpointStorageAccess |
Implementations of this interface should implement methods acting as an administration role for
checkpoint storage, which defined in
CheckpointStorageCoordinatorView . |
CheckpointStorageCoordinatorView |
This interface creates a
CheckpointStorageLocation to which an individual checkpoint or
savepoint is stored. |
CheckpointStorageFactory<T extends CheckpointStorage> |
A factory to create a specific
CheckpointStorage . |
CheckpointStorageLocation |
A storage location for one particular checkpoint, offering data persistent, metadata persistence,
and lifecycle/cleanup methods.
|
CheckpointStorageWorkerView |
This interface implements the durable storage of checkpoint data and metadata streams.
|
CheckpointStreamFactory |
A factory for checkpoint output streams, which are used to persist data for checkpoints.
|
CheckpointStreamWithResultProvider |
Interface that provides access to a CheckpointStateOutputStream and a method to provide the
SnapshotResult . |
CheckpointStreamWithResultProvider.KeyedStateHandleFactory |
Factory method for a
KeyedStateHandle to be used in CheckpointStreamWithResultProvider.toKeyedStateHandleSnapshotResult(SnapshotResult, KeyGroupRangeOffsets,
KeyedStateHandleFactory) . |
CompletedCheckpointStorageLocation |
The CompletedCheckpointStorageLocation describes the storage aspect of a completed checkpoint.
|
CompositeStateHandle |
Base of all snapshots that are taken by
StateBackend s and some other components in tasks. |
ConfigurableCheckpointStorage |
An interface for checkpoint storage types that pick up additional parameters from a
configuration.
|
ConfigurableStateBackend |
An interface for state backends that pick up additional parameters from a configuration.
|
FullSnapshotResources<K> |
A
SnapshotResources to be used with the backend-independent FullSnapshotAsyncWriter . |
FunctionInitializationContext |
This interface provides a context in which user functions can initialize by registering to
managed state (i.e.
|
FunctionSnapshotContext |
This interface provides a context in which user functions that use managed state (i.e.
|
IncrementalKeyedStateHandle |
Common interface to all incremental
KeyedStateHandle . |
InternalPriorityQueue<T> |
Interface for collection that gives in order access to elements w.r.t their priority.
|
IterableStateSnapshot<K,N,S> |
A
StateSnapshot that can return an iterator over all contained StateEntries . |
Keyed<K> |
Interface for objects that have a key attribute.
|
KeyedStateBackend<K> |
A keyed state backend provides methods for managing keyed state.
|
KeyedStateBackend.KeySelectionListener<K> |
Listener is given a callback when
KeyedStateBackend.setCurrentKey(K) is called (key context changes). |
KeyedStateFactory |
This factory produces concrete internal state objects.
|
KeyedStateFunction<K,S extends State> |
A function to be applied to all keyed states.
|
KeyedStateHandle |
Base for the handles of the checkpointed states in keyed streams.
|
KeyExtractorFunction<T> |
Function to extract a key from a given object.
|
KeyGroupedInternalPriorityQueue<T> |
This interface exists as (temporary) adapter between the new
InternalPriorityQueue and
the old way in which timers are written in a snapshot. |
KeyGroupPartitioner.ElementReaderFunction<T> |
This functional interface defines how one element is read from a
DataInputView . |
KeyGroupPartitioner.ElementWriterFunction<T> |
This functional interface defines how one element is written to a
DataOutputView . |
KeyGroupPartitioner.KeyGroupElementsConsumer<T> |
Functional interface to consume elements from a key group.
|
KeyGroupPartitioner.PartitioningResult<T> |
This represents the result of key-group partitioning.
|
KeyGroupsList |
This interface offers ordered random read access to multiple key group ids.
|
KeyValueStateIterator |
Iterator that over all key-value state entries in a
KeyedStateBackend . |
LocalRecoveryDirectoryProvider |
Provides directories for local recovery.
|
ManagedInitializationContext |
This interface provides a context in which operators can initialize by registering to managed
state (i.e.
|
ManagedSnapshotContext |
This interface provides a context in which operators that use managed state (i.e.
|
OperatorStateBackend |
Interface that combines both, the user facing
OperatorStateStore interface and the system
interface Snapshotable |
OperatorStateHandle |
Interface of a state handle for operator state.
|
OwnedTaskLocalStateStore |
This interface represents the administrative interface to
TaskLocalStateStore , that only
the owner of the object should see. |
PriorityComparable<T> |
Interface for objects that can be compared by priority.
|
PriorityComparator<T> |
This interface works similar to
Comparable and is used to prioritize between two objects. |
PriorityQueueSetFactory |
Factory for
KeyGroupedInternalPriorityQueue instances. |
RestoreOperation<R> |
Interface for restore operation.
|
RetrievableStateHandle<T extends Serializable> |
Handle to state that can be read back again via
RetrievableStateHandle.retrieveState() . |
SavepointKeyedStateHandle |
A
KeyedStateHandle that points to a savepoint taken in the unified format. |
SharedStateRegistry |
This registry manages state that is shared across (incremental) checkpoints, and is responsible
for deleting shared state that is no longer used in any valid checkpoint.
|
SharedStateRegistryFactory |
Simple factory to produce
SharedStateRegistry objects. |
Snapshotable<S extends StateObject> |
Interface for objects that can snapshot its state (state backends currently).
|
SnapshotResources |
A set of resources produced in the synchronous part of a snapshot required to finish the
snapshot.
|
SnapshotStrategy<S extends StateObject,SR extends SnapshotResources> |
Interface for different snapshot approaches in state backends.
|
SnapshotStrategy.SnapshotResultSupplier<S extends StateObject> |
A supplier for a
SnapshotResult with an access to a CloseableRegistry for io
tasks that need to be closed when cancelling the async part of the checkpoint. |
StateBackend |
A State Backend defines how the state of a streaming application is stored locally within
the cluster.
|
StateBackendBuilder<T,E extends Throwable> |
A mutable builder to build a state backend instance.
|
StateBackendFactory<T extends StateBackend> |
A factory to create a specific state backend.
|
StateEntry<K,N,S> |
Interface of entries in a state backend.
|
StateInitializationContext |
This interface provides a context in which operators can initialize by registering to managed
state (i.e.
|
StateObject |
Base of all handles that represent checkpointed state in some form.
|
StateSnapshot |
General interface for state snapshots that should be written partitioned by key-groups.
|
StateSnapshot.StateKeyGroupWriter |
Interface for writing a snapshot that is partitioned into key-groups.
|
StateSnapshotContext |
This interface provides a context in which operators that use managed (i.e.
|
StateSnapshotKeyGroupReader |
Interface for state de-serialization into
StateTable s by key-group. |
StateSnapshotRestore |
Interface to deal with state snapshot and restore of state.
|
StateSnapshotTransformer<T> |
Transformer of state values which are included or skipped in the snapshot.
|
StateSnapshotTransformer.CollectionStateSnapshotTransformer<T> |
Collection state specific transformer which says how to transform entries of the collection.
|
StateSnapshotTransformer.StateSnapshotTransformFactory<T> |
This factory creates state transformers depending on the form of values to transform.
|
StateTransformationFunction<S,T> |
Interface for a binary function that is used for push-down of state transformation into state
backends.
|
StreamStateHandle |
A
StateObject that represents state that was written to a stream. |
TaskLocalStateStore |
Classes that implement this interface serve as a task-manager-level local storage for local
checkpointed state.
|
TaskStateManager |
This interface provides methods to report and retrieve state for a task.
|
TestableKeyedStateBackend<K> |
A keyed state backend interface for internal testing purpose.
|
Class | Description |
---|---|
AbstractChannelStateHandle<Info> |
Abstract channel state handle.
|
AbstractChannelStateHandle.StateContentMetaInfo |
Describes the underlying content.
|
AbstractKeyedStateBackend<K> |
Base implementation of KeyedStateBackend.
|
AbstractKeyedStateBackendBuilder<K> |
An abstract base implementation of the
StateBackendBuilder interface. |
AbstractManagedMemoryStateBackend |
Abstract base class for state backends that use managed memory.
|
AbstractStateBackend |
An abstract base implementation of the
StateBackend interface. |
ArrayListSerializer<T> | |
ArrayListSerializerSnapshot<T> |
Snapshot class for the
ArrayListSerializer . |
AsyncSnapshotCallable<T> |
Base class that outlines the strategy for asynchronous snapshots.
|
BoundedInputStream |
Wrapper around a FSDataInputStream to limit the maximum read offset.
|
ChainedStateHandle<T extends StateObject> |
Handle to state handles for the operators in an operator chain.
|
ChangelogTaskLocalStateStore |
Changelog's implementation of a
TaskLocalStateStore . |
CheckpointMetadataOutputStream |
An output stream for checkpoint metadata.
|
CheckpointStateOutputStream |
A dedicated output stream that produces a
StreamStateHandle when closed. |
CheckpointStorageLoader |
This class contains utility methods to load checkpoint storage from configurations.
|
CheckpointStorageLocationReference |
A reference to a storage location.
|
CheckpointStreamWithResultProvider.PrimaryAndSecondaryStream |
Implementation of
CheckpointStreamWithResultProvider that creates both, the
primary/remote/jm-owned state and the secondary/local/tm-owned state. |
CheckpointStreamWithResultProvider.PrimaryStreamOnly |
Implementation of
CheckpointStreamWithResultProvider that only creates the
primary/remote/jm-owned state. |
CompositeKeySerializationUtils |
Utils for RocksDB state serialization and deserialization.
|
DefaultKeyedStateStore |
Default implementation of KeyedStateStore that currently forwards state registration to a
RuntimeContext . |
DefaultOperatorStateBackend |
Default implementation of OperatorStateStore that provides the ability to make snapshots.
|
DefaultOperatorStateBackendBuilder |
Builder class for
DefaultOperatorStateBackend which handles all necessary initializations
and clean ups. |
DirectoryKeyedStateHandle |
This class is a keyed state handle based on a directory.
|
DirectoryStateHandle |
This state handle represents a directory.
|
DoneFuture<T> |
A
Future that is always done and will just yield the object that was given at creation
time. |
DuplicatingCheckpointOutputStream |
A CheckpointStateOutputStream that wraps a primary and a secondary CheckpointStateOutputStream
and duplicates all writes into both streams.
|
FullSnapshotAsyncWriter<K> |
An asynchronous writer that can write a full snapshot/savepoint from a
FullSnapshotResources . |
FullSnapshotUtil |
Utility methods and constants around creating and restoring full snapshots using
FullSnapshotAsyncWriter . |
HeapBroadcastState<K,V> |
A
Broadcast State backed a heap-based Map . |
HeapPriorityQueuesManager |
Manages creating heap priority queues along with their counterpart meta info.
|
IncrementalKeyedStateHandle.HandleAndLocalPath |
A Holder of StreamStateHandle and the corresponding localPath.
|
IncrementalLocalKeyedStateHandle |
State handle for local copies of
IncrementalRemoteKeyedStateHandle . |
IncrementalRemoteKeyedStateHandle |
The handle to states of an incremental snapshot.
|
InputChannelStateHandle |
Handle to an InputChannel state. |
JavaSerializer<T extends Serializable> |
A
TypeSerializer that uses Java serialization. |
JavaSerializer.JavaSerializerSnapshot<T extends Serializable> |
Serializer configuration snapshot for compatibility and format evolution.
|
KeyedBackendSerializationProxy<K> |
Serialization proxy for all meta data in keyed state backends.
|
KeyedStateCheckpointOutputStream |
Checkpoint output stream that allows to write raw keyed state in a partitioned way, split into
key-groups.
|
KeyGroupPartitioner<T> |
Class that contains the base algorithm for partitioning data into key-groups.
|
KeyGroupRange |
This class defines a range of key-group indexes.
|
KeyGroupRangeAssignment | |
KeyGroupRangeOffsets |
This class combines a key-group range with offsets that correspond to the key-groups in the
range.
|
KeyGroupsSavepointStateHandle |
A
KeyGroupsStateHandle that describes a savepoint in the unified format. |
KeyGroupsStateHandle |
A handle to the partitioned stream operator state after it has been checkpointed.
|
KeyGroupStatePartitionStreamProvider |
This class provides access to an input stream that contains state data for one key group and the
key group id.
|
ListDelimitedSerializer |
Encapsulates a logic of serialization and deserialization of a list with a delimiter.
|
LocalRecoveryConfig |
This class encapsulates the completed configuration for local recovery, i.e.
|
LocalRecoveryDirectoryProviderImpl |
Implementation of
LocalRecoveryDirectoryProvider . |
NonClosingCheckpointOutputStream<T extends StreamStateHandle> |
Abstract class to implement custom checkpoint output streams which should not be closable for
user code.
|
NoOpTaskLocalStateStoreImpl |
This class implements a
TaskLocalStateStore with no functionality and is used when local
recovery is disabled. |
NotDuplicatingCheckpointStateToolset |
An empty implementation of
CheckpointStateToolset . |
OperatorBackendSerializationProxy |
Serialization proxy for all meta data in operator state backends.
|
OperatorStateCheckpointOutputStream |
Checkpoint output stream that allows to write raw operator state in a partitioned way.
|
OperatorStateHandle.StateMetaInfo |
Meta information about the operator state handle.
|
OperatorStateRestoreOperation |
Implementation of operator state restore operation.
|
OperatorStreamStateHandle |
State handle for partitionable operator state.
|
PartitionableListState<S> |
Implementation of operator list state.
|
PhysicalStateHandleID |
Unique ID that allows for physical comparison between state handles.
|
PlaceholderStreamStateHandle |
A placeholder state handle for shared state that will replaced by an original that was created in
a previous checkpoint.
|
RegisteredBroadcastStateBackendMetaInfo<K,V> | |
RegisteredKeyValueStateBackendMetaInfo<N,S> |
Compound meta information for a registered state in a keyed state backend.
|
RegisteredOperatorStateBackendMetaInfo<S> |
Compound meta information for a registered state in an operator state backend.
|
RegisteredPriorityQueueStateBackendMetaInfo<T> |
Meta information about a priority queue state in a backend.
|
RegisteredStateMetaInfoBase |
Base class for all registered state in state backends.
|
ResultSubpartitionStateHandle |
Handle to a ResultSubpartition state. |
RetrievableStreamStateHandle<T extends Serializable> |
Wrapper around a
StreamStateHandle to make the referenced state object retrievable trough
a simple get call. |
SavepointResources<K> |
Savepoint resources for a
KeyedStateBackend . |
SavepointSnapshotStrategy<K> |
A
SnapshotStrategy that produces unified savepoints. |
SerializedCompositeKeyBuilder<K> |
Responsible for serialization of currentKey, currentGroup and namespace.
|
SharedStateRegistryImpl |
SharedStateRegistry implementation. |
SharedStateRegistryImpl.EmptyDiscardStateObjectForRegister |
An object with empty discardState for registering.
|
SharedStateRegistryKey |
This class represents a key that uniquely identifies (on a logical level) state handles for
registration in the
SharedStateRegistry . |
SnappyStreamCompressionDecorator |
This implementation decorates the stream with snappy compression.
|
SnapshotDirectory |
This class represents a directory that is the target for a state snapshot.
|
SnapshotResult<T extends StateObject> |
This class contains the combined results from the snapshot of a state backend:
A state object representing the state that will be reported to the Job Manager to
acknowledge the checkpoint.
|
SnapshotStrategyRunner<T extends StateObject,SR extends SnapshotResources> |
A class to execute a
SnapshotStrategy . |
StateBackendLoader |
This class contains utility methods to load state backends from configurations.
|
StateEntry.SimpleStateEntry<K,N,S> | |
StateHandleID |
Unique ID that allows for logical comparison between state handles.
|
StateInitializationContextImpl |
Default implementation of
StateInitializationContext . |
StatePartitionStreamProvider |
This class provides access to input streams that contain data of one state partition of a
partitionable state.
|
StateSerializerProvider<T> |
A
StateSerializerProvider wraps logic on how to obtain serializers for registered state,
either with the previous schema of state in checkpoints or the current schema of state. |
StateSnapshotContextSynchronousImpl |
This class is a default implementation for StateSnapshotContext.
|
StateSnapshotTransformers |
Collection of common state snapshot transformers and their factories.
|
StateSnapshotTransformers.ListStateSnapshotTransformer<T> |
General implementation of list state transformer.
|
StateSnapshotTransformers.ListStateSnapshotTransformFactory<T> | |
StateSnapshotTransformers.MapStateSnapshotTransformer<K,V> |
General implementation of map state transformer.
|
StateSnapshotTransformers.MapStateSnapshotTransformFactory<K,V> | |
StateSnapshotTransformers.StateSnapshotTransformFactoryWrapAdaptor<S,T> | |
StateUtil |
Helpers for
StateObject related code. |
StreamCompressionDecorator |
Implementations of this interface decorate streams with a compression scheme.
|
TaskExecutorLocalStateStoresManager |
This class holds the all
TaskLocalStateStoreImpl objects for a task executor (manager). |
TaskExecutorStateChangelogStoragesManager |
This class holds the all
StateChangelogStorage objects for a task executor (manager). |
TaskLocalStateStoreImpl |
Main implementation of a
TaskLocalStateStore . |
TaskStateManagerImpl |
This class is the default implementation of
TaskStateManager and collaborates with the
job manager through CheckpointResponder ) as well as a task-manager-local state store. |
UncompressedStreamCompressionDecorator |
This implementation does not decorate the stream with any compression.
|
VoidNamespace |
Singleton placeholder class for state without a namespace.
|
VoidNamespaceSerializer |
Serializer for
VoidNamespace . |
VoidNamespaceSerializer.VoidNamespaceSerializerSnapshot |
Serializer configuration snapshot for compatibility and format evolution.
|
VoidNamespaceTypeInfo |
Enum | Description |
---|---|
CheckpointedStateScope |
The scope for a chunk of checkpointed state.
|
OperatorStateHandle.Mode |
The modes that determine how an
OperatorStreamStateHandle is assigned to tasks during
restore. |
SnapshotExecutionType |
Determines how the write-part of a snapshot should be executed.
|
StateSnapshotTransformer.CollectionStateSnapshotTransformer.TransformStrategy |
Exception | Description |
---|---|
BackendBuildingException |
Exceptions which indicate that a state backend building has failed.
|
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.