@PublicEvolving public class MemoryStateBackend extends AbstractFileStateBackend implements ConfigurableStateBackend
This state backend should be used only for experimentation, quick local setups,
or for streaming applications that have very small state: Because it requires checkpoints to
go through the JobManager's memory, larger state will occupy larger portions of the JobManager's
main memory, reducing operational stability.
For any other setup, the FsStateBackend
should be used. The FsStateBackend
holds the working state on the TaskManagers in the same way, but
checkpoints state directly to files rather then to the JobManager's memory, thus supporting
large state sizes.
State checkpointing with this state backend is subject to the following conditions:
getMaxStateSize()
.For the use cases where the state sizes can be handled by this backend, the backend does guarantee persistence for savepoints, externalized checkpoints (of configured), and checkpoints (when high-availability is configured).
As for all state backends, this backend can either be configured within the application (by creating the backend with the respective constructor parameters and setting it on the execution environment) or by specifying it in the Flink configuration.
If the state backend was specified in the application, it may pick up additional configuration
parameters from the Flink configuration. For example, if the backend if configured in the application
without a default savepoint directory, it will pick up a default savepoint directory specified in the
Flink configuration of the running job/cluster. That behavior is implemented via the
configure(Configuration)
method.
Modifier and Type | Field and Description |
---|---|
static int |
DEFAULT_MAX_STATE_SIZE
The default maximal size that the snapshotted memory state may have (5 MiBytes).
|
Constructor and Description |
---|
MemoryStateBackend()
Creates a new memory state backend that accepts states whose serialized forms are
up to the default state size (5 MB).
|
MemoryStateBackend(boolean asynchronousSnapshots)
Creates a new memory state backend that accepts states whose serialized forms are
up to the default state size (5 MB).
|
MemoryStateBackend(int maxStateSize)
Creates a new memory state backend that accepts states whose serialized forms are
up to the given number of bytes.
|
MemoryStateBackend(int maxStateSize,
boolean asynchronousSnapshots)
Creates a new memory state backend that accepts states whose serialized forms are
up to the given number of bytes and that uses asynchronous snashots as configured.
|
MemoryStateBackend(String checkpointPath,
String savepointPath)
Creates a new MemoryStateBackend, setting optionally the path to persist checkpoint metadata
to, and to persist savepoints to.
|
MemoryStateBackend(String checkpointPath,
String savepointPath,
int maxStateSize,
TernaryBoolean asynchronousSnapshots)
Creates a new MemoryStateBackend, setting optionally the paths to persist checkpoint metadata
and savepoints to, as well as configuring state thresholds and asynchronous operations.
|
Modifier and Type | Method and Description |
---|---|
MemoryStateBackend |
configure(Configuration config)
Creates a copy of this state backend that uses the values defined in the configuration
for fields where that were not specified in this state backend.
|
CheckpointStorage |
createCheckpointStorage(JobID jobId)
Creates a storage for checkpoints for the given job.
|
<K> AbstractKeyedStateBackend<K> |
createKeyedStateBackend(Environment env,
JobID jobID,
String operatorIdentifier,
TypeSerializer<K> keySerializer,
int numberOfKeyGroups,
KeyGroupRange keyGroupRange,
TaskKvStateRegistry kvStateRegistry,
TtlTimeProvider ttlTimeProvider)
Creates a new
AbstractKeyedStateBackend that is responsible for holding keyed state
and checkpointing it. |
OperatorStateBackend |
createOperatorStateBackend(Environment env,
String operatorIdentifier)
Creates a new
OperatorStateBackend that can be used for storing operator state. |
int |
getMaxStateSize()
Gets the maximum size that an individual state can have, as configured in the
constructor (by default 5242880).
|
boolean |
isUsingAsynchronousSnapshots()
Gets whether the key/value data structures are asynchronously snapshotted.
|
String |
toString() |
getCheckpointPath, getSavepointPath, resolveCheckpoint
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
createKeyedStateBackend
public static final int DEFAULT_MAX_STATE_SIZE
public MemoryStateBackend()
Checkpoint and default savepoint locations are used as specified in the runtime configuration.
public MemoryStateBackend(boolean asynchronousSnapshots)
Checkpoint and default savepoint locations are used as specified in the runtime configuration.
asynchronousSnapshots
- Switch to enable asynchronous snapshots.public MemoryStateBackend(int maxStateSize)
Checkpoint and default savepoint locations are used as specified in the runtime configuration.
WARNING: Increasing the size of this value beyond the default value (5242880) should be done with care. The checkpointed state needs to be send to the JobManager via limited size RPC messages, and there and the JobManager needs to be able to hold all aggregated state in its memory.
maxStateSize
- The maximal size of the serialized statepublic MemoryStateBackend(int maxStateSize, boolean asynchronousSnapshots)
Checkpoint and default savepoint locations are used as specified in the runtime configuration.
WARNING: Increasing the size of this value beyond the default value (5242880) should be done with care. The checkpointed state needs to be send to the JobManager via limited size RPC messages, and there and the JobManager needs to be able to hold all aggregated state in its memory.
maxStateSize
- The maximal size of the serialized stateasynchronousSnapshots
- Switch to enable asynchronous snapshots.public MemoryStateBackend(@Nullable String checkpointPath, @Nullable String savepointPath)
checkpointPath
- The path to write checkpoint metadata to. If null, the value from
the runtime configuration will be used.savepointPath
- The path to write savepoints to. If null, the value from
the runtime configuration will be used.public MemoryStateBackend(@Nullable String checkpointPath, @Nullable String savepointPath, int maxStateSize, TernaryBoolean asynchronousSnapshots)
WARNING: Increasing the size of this value beyond the default value (5242880) should be done with care. The checkpointed state needs to be send to the JobManager via limited size RPC messages, and there and the JobManager needs to be able to hold all aggregated state in its memory.
checkpointPath
- The path to write checkpoint metadata to. If null, the value from
the runtime configuration will be used.savepointPath
- The path to write savepoints to. If null, the value from
the runtime configuration will be used.maxStateSize
- The maximal size of the serialized state.asynchronousSnapshots
- Flag to switch between synchronous and asynchronous
snapshot mode. If null, the value configured in the
runtime configuration will be used.public int getMaxStateSize()
public boolean isUsingAsynchronousSnapshots()
If not explicitly configured, this is the default value of
CheckpointingOptions.ASYNC_SNAPSHOTS
.
public MemoryStateBackend configure(Configuration config)
configure
in interface ConfigurableStateBackend
config
- the configurationpublic CheckpointStorage createCheckpointStorage(JobID jobId) throws IOException
StateBackend
createCheckpointStorage
in interface StateBackend
jobId
- The job to store checkpoint data for.IOException
- Thrown if the checkpoint storage cannot be initialized.public OperatorStateBackend createOperatorStateBackend(Environment env, String operatorIdentifier) throws Exception
StateBackend
OperatorStateBackend
that can be used for storing operator state.
Operator state is state that is associated with parallel operator (or function) instances, rather than with keys.
createOperatorStateBackend
in interface StateBackend
createOperatorStateBackend
in class AbstractStateBackend
env
- The runtime environment of the executing task.operatorIdentifier
- The identifier of the operator whose state should be stored.Exception
- This method may forward all exceptions that occur while instantiating the backend.public <K> AbstractKeyedStateBackend<K> createKeyedStateBackend(Environment env, JobID jobID, String operatorIdentifier, TypeSerializer<K> keySerializer, int numberOfKeyGroups, KeyGroupRange keyGroupRange, TaskKvStateRegistry kvStateRegistry, TtlTimeProvider ttlTimeProvider)
StateBackend
AbstractKeyedStateBackend
that is responsible for holding keyed state
and checkpointing it.
Keyed State is state where each value is bound to a key.
createKeyedStateBackend
in interface StateBackend
createKeyedStateBackend
in class AbstractStateBackend
K
- The type of the keys by which the state is organized.Copyright © 2014–2020 The Apache Software Foundation. All rights reserved.