public interface ExecutionGraph extends AccessExecutionGraph
The execution graph consists of the following constructs:
ExecutionJobVertex
represents one vertex from the JobGraph (usually one
operation like "map" or "join") during execution. It holds the aggregated state of all
parallel subtasks. The ExecutionJobVertex is identified inside the graph by the JobVertexID
, which it takes from the JobGraph's corresponding JobVertex.
ExecutionVertex
represents one parallel subtask. For each ExecutionJobVertex,
there are as many ExecutionVertices as the parallelism. The ExecutionVertex is identified
by the ExecutionJobVertex and the index of the parallel subtask
Execution
is one attempt to execute a ExecutionVertex. There may be multiple
Executions for the ExecutionVertex, in case of a failure, or in the case where some data
needs to be recomputed because it is no longer available when requested by later
operations. An Execution is always identified by an ExecutionAttemptID
. All
messages between the JobManager and the TaskManager about deployment of tasks and updates
in the task status always use the ExecutionAttemptID to address the message receiver.
Modifier and Type | Method and Description |
---|---|
Map<String,OptionalFailure<Accumulator<?,?>>> |
aggregateUserAccumulators()
Merges all accumulator results from the tasks previously executed in the Executions.
|
void |
attachJobGraph(List<JobVertex> topologicallySorted) |
void |
cancel() |
void |
enableCheckpointing(CheckpointCoordinatorConfiguration chkConfig,
List<MasterTriggerRestoreHook<?>> masterHooks,
CheckpointIDCounter checkpointIDCounter,
CompletedCheckpointStore checkpointStore,
StateBackend checkpointStateBackend,
CheckpointStorage checkpointStorage,
CheckpointStatsTracker statsTracker,
CheckpointsCleaner checkpointsCleaner,
String changelogStorage) |
void |
failJob(Throwable cause,
long timestamp) |
Optional<AccessExecution> |
findExecution(ExecutionAttemptID attemptId) |
Optional<String> |
findVertexWithAttempt(ExecutionAttemptID attemptId) |
Iterable<ExecutionVertex> |
getAllExecutionVertices()
Returns an iterable containing all execution vertices for this execution graph.
|
Map<IntermediateDataSetID,IntermediateResult> |
getAllIntermediateResults() |
Map<JobVertexID,ExecutionJobVertex> |
getAllVertices()
Returns a map containing all job vertices for this execution graph.
|
CheckpointCoordinator |
getCheckpointCoordinator() |
Throwable |
getFailureCause() |
Configuration |
getJobConfiguration() |
ComponentMainThreadExecutor |
getJobMasterMainThreadExecutor() |
ExecutionJobVertex |
getJobVertex(JobVertexID id)
Returns the job vertex for the given
JobVertexID . |
KvStateLocationRegistry |
getKvStateLocationRegistry() |
long |
getNumberOfRestarts()
Gets the number of restarts, including full restarts and fine grained restarts.
|
int |
getNumFinishedVertices() |
Map<ExecutionAttemptID,Execution> |
getRegisteredExecutions() |
ResultPartitionAvailabilityChecker |
getResultPartitionAvailabilityChecker() |
IntermediateResultPartition |
getResultPartitionOrThrow(IntermediateResultPartitionID id)
Gets the intermediate result partition by the given partition ID, or throw an exception if
the partition is not found.
|
SchedulingTopology |
getSchedulingTopology() |
CompletableFuture<JobStatus> |
getTerminationFuture()
Returns the termination future of this
ExecutionGraph . |
Iterable<ExecutionJobVertex> |
getVerticesTopologically()
Returns an iterable containing all job vertices for this execution graph in the order they
were created.
|
void |
incrementRestarts() |
void |
initFailureCause(Throwable t,
long timestamp) |
default void |
initializeJobVertex(ExecutionJobVertex ejv,
long createTimestamp) |
void |
initializeJobVertex(ExecutionJobVertex ejv,
long createTimestamp,
Map<IntermediateDataSetID,JobVertexInputInfo> jobVertexInputInfos)
Initialize the given execution job vertex, mainly includes creating execution vertices
according to the parallelism, and connecting to the predecessors.
|
void |
notifyNewlyInitializedJobVertices(List<ExecutionJobVertex> vertices)
Notify that some job vertices have been newly initialized, execution graph will try to update
scheduling topology.
|
void |
registerJobStatusListener(JobStatusListener listener) |
void |
setInternalTaskFailuresListener(InternalFailuresListener internalTaskFailuresListener) |
void |
setJsonPlan(String jsonPlan) |
void |
start(ComponentMainThreadExecutor jobMasterMainThreadExecutor) |
void |
suspend(Throwable suspensionCause)
Suspends the current ExecutionGraph.
|
boolean |
transitionState(JobStatus current,
JobStatus newState) |
void |
transitionToRunning() |
void |
updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
Updates the accumulators during the runtime of a job.
|
boolean |
updateState(TaskExecutionStateTransition state)
Updates the state of one of the ExecutionVertex's Execution attempts.
|
JobStatus |
waitUntilTerminal() |
getAccumulatorResultsStringified, getAccumulatorsSerialized, getArchivedExecutionConfig, getChangelogStorageName, getCheckpointCoordinatorConfiguration, getCheckpointStatsSnapshot, getCheckpointStorageName, getFailureInfo, getJobID, getJobName, getJsonPlan, getState, getStateBackendName, getStatusTimestamp, isChangelogStateBackendEnabled, isStoppable
void start(@Nonnull ComponentMainThreadExecutor jobMasterMainThreadExecutor)
SchedulingTopology getSchedulingTopology()
void enableCheckpointing(CheckpointCoordinatorConfiguration chkConfig, List<MasterTriggerRestoreHook<?>> masterHooks, CheckpointIDCounter checkpointIDCounter, CompletedCheckpointStore checkpointStore, StateBackend checkpointStateBackend, CheckpointStorage checkpointStorage, CheckpointStatsTracker statsTracker, CheckpointsCleaner checkpointsCleaner, String changelogStorage)
@Nullable CheckpointCoordinator getCheckpointCoordinator()
KvStateLocationRegistry getKvStateLocationRegistry()
void setJsonPlan(String jsonPlan)
Configuration getJobConfiguration()
Throwable getFailureCause()
Iterable<ExecutionJobVertex> getVerticesTopologically()
AccessExecutionGraph
getVerticesTopologically
in interface AccessExecutionGraph
Iterable<ExecutionVertex> getAllExecutionVertices()
AccessExecutionGraph
getAllExecutionVertices
in interface AccessExecutionGraph
ExecutionJobVertex getJobVertex(JobVertexID id)
AccessExecutionGraph
JobVertexID
.getJobVertex
in interface AccessExecutionGraph
id
- id of job vertex to be returnednull
Map<JobVertexID,ExecutionJobVertex> getAllVertices()
AccessExecutionGraph
getAllVertices
in interface AccessExecutionGraph
long getNumberOfRestarts()
Map<IntermediateDataSetID,IntermediateResult> getAllIntermediateResults()
IntermediateResultPartition getResultPartitionOrThrow(IntermediateResultPartitionID id)
id
- of the intermediate result partitionMap<String,OptionalFailure<Accumulator<?,?>>> aggregateUserAccumulators()
void updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
accumulatorSnapshot
- The serialized flink and user-defined accumulatorsvoid setInternalTaskFailuresListener(InternalFailuresListener internalTaskFailuresListener)
void attachJobGraph(List<JobVertex> topologicallySorted) throws JobException
JobException
void transitionToRunning()
void cancel()
void suspend(Throwable suspensionCause)
The JobStatus will be directly set to JobStatus.SUSPENDED
iff the current state is
not a terminal state. All ExecutionJobVertices will be canceled and the onTerminalState() is
executed.
The JobStatus.SUSPENDED
state is a local terminal state which stops the execution
of the job but does not remove the job from the HA job store so that it can be recovered by
another JobManager.
suspensionCause
- Cause of the suspensionvoid failJob(Throwable cause, long timestamp)
CompletableFuture<JobStatus> getTerminationFuture()
ExecutionGraph
. The termination future is
completed with the terminal JobStatus
once the ExecutionGraph reaches this terminal
state and all Execution
have been terminated.ExecutionGraph
.@VisibleForTesting JobStatus waitUntilTerminal() throws InterruptedException
InterruptedException
void incrementRestarts()
void initFailureCause(Throwable t, long timestamp)
boolean updateState(TaskExecutionStateTransition state)
state
- The state update.Map<ExecutionAttemptID,Execution> getRegisteredExecutions()
void registerJobStatusListener(JobStatusListener listener)
ResultPartitionAvailabilityChecker getResultPartitionAvailabilityChecker()
int getNumFinishedVertices()
@Nonnull ComponentMainThreadExecutor getJobMasterMainThreadExecutor()
default void initializeJobVertex(ExecutionJobVertex ejv, long createTimestamp) throws JobException
JobException
void initializeJobVertex(ExecutionJobVertex ejv, long createTimestamp, Map<IntermediateDataSetID,JobVertexInputInfo> jobVertexInputInfos) throws JobException
ejv
- The execution job vertex that needs to be initialized.createTimestamp
- The timestamp for creating execution vertices, used to initialize the
first Execution with.jobVertexInputInfos
- The input infos of this job vertex.JobException
void notifyNewlyInitializedJobVertices(List<ExecutionJobVertex> vertices)
vertices
- The execution job vertices that are newly initialized.Optional<String> findVertexWithAttempt(ExecutionAttemptID attemptId)
Optional<AccessExecution> findExecution(ExecutionAttemptID attemptId)
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.