public interface TaskStateManager extends CheckpointListener, AutoCloseable
When a checkpoint or savepoint is triggered on a task, it will create snapshots for all stream operator instances it owns. All operator snapshots from the task are then reported via this interface. A typical implementation will dispatch and forward the reported state information to interested parties such as the checkpoint coordinator or a local state store.
This interface also offers the complementary method that provides access to previously saved state of operator instances in the task for restore purposes.
Modifier and Type | Method and Description |
---|---|
LocalRecoveryConfig |
createLocalRecoveryConfig()
Returns the configuration for local recovery, i.e. the base directories for all file-based
local state of the owning subtask and the general mode for local recovery.
|
InflightDataRescalingDescriptor |
getInputRescalingDescriptor() |
InflightDataRescalingDescriptor |
getOutputRescalingDescriptor() |
Optional<Long> |
getRestoreCheckpointId()
Acquires the checkpoint id to restore from.
|
SequentialChannelStateReader |
getSequentialChannelStateReader() |
StateChangelogStorage<?> |
getStateChangelogStorage()
Returns the configured state changelog storage for this task.
|
StateChangelogStorageView<?> |
getStateChangelogStorageView(Configuration configuration,
ChangelogStateHandle changelogStateHandle)
Returns the state changelog storage view of given
ChangelogStateHandle for this task. |
boolean |
isTaskDeployedAsFinished()
Whether all the operators of the task are finished on restore.
|
PrioritizedOperatorSubtaskState |
prioritizedOperatorState(OperatorID operatorID)
Returns means to restore previously reported state of an operator running in the owning task.
|
void |
reportIncompleteTaskStateSnapshots(CheckpointMetaData checkpointMetaData,
CheckpointMetrics checkpointMetrics)
Report the stats for state snapshots for an aborted checkpoint.
|
void |
reportTaskStateSnapshots(CheckpointMetaData checkpointMetaData,
CheckpointMetrics checkpointMetrics,
TaskStateSnapshot acknowledgedState,
TaskStateSnapshot localState)
Report the state snapshots for the operator instances running in the owning task.
|
notifyCheckpointAborted, notifyCheckpointComplete
close
void reportTaskStateSnapshots(@Nonnull CheckpointMetaData checkpointMetaData, @Nonnull CheckpointMetrics checkpointMetrics, @Nullable TaskStateSnapshot acknowledgedState, @Nullable TaskStateSnapshot localState)
checkpointMetaData
- meta data from the checkpoint request.checkpointMetrics
- task level metrics for the checkpoint.acknowledgedState
- the reported states to acknowledge to the job manager.localState
- the reported states for local recovery.InflightDataRescalingDescriptor getInputRescalingDescriptor()
InflightDataRescalingDescriptor getOutputRescalingDescriptor()
void reportIncompleteTaskStateSnapshots(CheckpointMetaData checkpointMetaData, CheckpointMetrics checkpointMetrics)
checkpointMetaData
- meta data from the checkpoint request.checkpointMetrics
- task level metrics for the checkpoint.boolean isTaskDeployedAsFinished()
Optional<Long> getRestoreCheckpointId()
@Nonnull PrioritizedOperatorSubtaskState prioritizedOperatorState(OperatorID operatorID)
operatorID
- the id of the operator for which we request state.@Nonnull LocalRecoveryConfig createLocalRecoveryConfig()
SequentialChannelStateReader getSequentialChannelStateReader()
@Nullable StateChangelogStorage<?> getStateChangelogStorage()
@Nullable StateChangelogStorageView<?> getStateChangelogStorageView(Configuration configuration, ChangelogStateHandle changelogStateHandle)
ChangelogStateHandle
for this task.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.