public class DefaultExecutionGraph extends Object implements ExecutionGraph, InternalExecutionGraphAccessor
ExecutionGraph
.Constructor and Description |
---|
DefaultExecutionGraph(JobInformation jobInformation,
ScheduledExecutorService futureExecutor,
Executor ioExecutor,
Time rpcTimeout,
int executionHistorySizeLimit,
ClassLoader userClassLoader,
BlobWriter blobWriter,
PartitionGroupReleaseStrategy.Factory partitionGroupReleaseStrategyFactory,
ShuffleMaster<?> shuffleMaster,
JobMasterPartitionTracker partitionTracker,
TaskDeploymentDescriptorFactory.PartitionLocationConstraint partitionLocationConstraint,
ExecutionDeploymentListener executionDeploymentListener,
ExecutionStateUpdateListener executionStateUpdateListener,
long initializationTimestamp,
VertexAttemptNumberStore initialAttemptCounts,
VertexParallelismStore vertexParallelismStore,
boolean isDynamic,
ExecutionJobVertex.Factory executionJobVertexFactory,
List<JobStatusHook> jobStatusHooks) |
public DefaultExecutionGraph(JobInformation jobInformation, ScheduledExecutorService futureExecutor, Executor ioExecutor, Time rpcTimeout, int executionHistorySizeLimit, ClassLoader userClassLoader, BlobWriter blobWriter, PartitionGroupReleaseStrategy.Factory partitionGroupReleaseStrategyFactory, ShuffleMaster<?> shuffleMaster, JobMasterPartitionTracker partitionTracker, TaskDeploymentDescriptorFactory.PartitionLocationConstraint partitionLocationConstraint, ExecutionDeploymentListener executionDeploymentListener, ExecutionStateUpdateListener executionStateUpdateListener, long initializationTimestamp, VertexAttemptNumberStore initialAttemptCounts, VertexParallelismStore vertexParallelismStore, boolean isDynamic, ExecutionJobVertex.Factory executionJobVertexFactory, List<JobStatusHook> jobStatusHooks) throws IOException
IOException
public void start(@Nonnull ComponentMainThreadExecutor jobMasterMainThreadExecutor)
start
in interface ExecutionGraph
public SchedulingTopology getSchedulingTopology()
getSchedulingTopology
in interface ExecutionGraph
public TaskDeploymentDescriptorFactory.PartitionLocationConstraint getPartitionLocationConstraint()
getPartitionLocationConstraint
in interface InternalExecutionGraphAccessor
@Nonnull public ComponentMainThreadExecutor getJobMasterMainThreadExecutor()
getJobMasterMainThreadExecutor
in interface ExecutionGraph
getJobMasterMainThreadExecutor
in interface InternalExecutionGraphAccessor
public TernaryBoolean isChangelogStateBackendEnabled()
AccessExecutionGraph
isChangelogStateBackendEnabled
in interface AccessExecutionGraph
public Optional<String> getStateBackendName()
AccessExecutionGraph
getStateBackendName
in interface AccessExecutionGraph
public Optional<String> getCheckpointStorageName()
AccessExecutionGraph
getCheckpointStorageName
in interface AccessExecutionGraph
public Optional<String> getChangelogStorageName()
AccessExecutionGraph
getChangelogStorageName
in interface AccessExecutionGraph
public void enableCheckpointing(CheckpointCoordinatorConfiguration chkConfig, List<MasterTriggerRestoreHook<?>> masterHooks, CheckpointIDCounter checkpointIDCounter, CompletedCheckpointStore checkpointStore, StateBackend checkpointStateBackend, CheckpointStorage checkpointStorage, CheckpointStatsTracker statsTracker, CheckpointsCleaner checkpointsCleaner, String changelogStorageName)
enableCheckpointing
in interface ExecutionGraph
@Nullable public CheckpointCoordinator getCheckpointCoordinator()
getCheckpointCoordinator
in interface ExecutionGraph
public KvStateLocationRegistry getKvStateLocationRegistry()
getKvStateLocationRegistry
in interface ExecutionGraph
public CheckpointCoordinatorConfiguration getCheckpointCoordinatorConfiguration()
AccessExecutionGraph
CheckpointCoordinatorConfiguration
or null
if checkpointing
is disabled.getCheckpointCoordinatorConfiguration
in interface AccessExecutionGraph
public CheckpointStatsSnapshot getCheckpointStatsSnapshot()
AccessExecutionGraph
null
if checkpointing is
disabled.getCheckpointStatsSnapshot
in interface AccessExecutionGraph
public void setJsonPlan(String jsonPlan)
setJsonPlan
in interface ExecutionGraph
public String getJsonPlan()
AccessExecutionGraph
getJsonPlan
in interface AccessExecutionGraph
public Either<SerializedValue<JobInformation>,PermanentBlobKey> getJobInformationOrBlobKey()
getJobInformationOrBlobKey
in interface InternalExecutionGraphAccessor
public JobID getJobID()
AccessExecutionGraph
JobID
for this execution graph.getJobID
in interface AccessExecutionGraph
getJobID
in interface InternalExecutionGraphAccessor
public String getJobName()
AccessExecutionGraph
getJobName
in interface AccessExecutionGraph
public boolean isStoppable()
AccessExecutionGraph
isStoppable
in interface AccessExecutionGraph
public Configuration getJobConfiguration()
getJobConfiguration
in interface ExecutionGraph
public ClassLoader getUserClassLoader()
getUserClassLoader
in interface InternalExecutionGraphAccessor
public JobStatus getState()
AccessExecutionGraph
JobStatus
for this execution graph.getState
in interface AccessExecutionGraph
getState
in interface JobStatusProvider
public Throwable getFailureCause()
getFailureCause
in interface ExecutionGraph
public ErrorInfo getFailureInfo()
AccessExecutionGraph
getFailureInfo
in interface AccessExecutionGraph
public long getNumberOfRestarts()
ExecutionGraph
getNumberOfRestarts
in interface ExecutionGraph
public int getNumFinishedVertices()
getNumFinishedVertices
in interface ExecutionGraph
public ExecutionJobVertex getJobVertex(JobVertexID id)
AccessExecutionGraph
JobVertexID
.getJobVertex
in interface AccessExecutionGraph
getJobVertex
in interface ExecutionGraph
getJobVertex
in interface InternalExecutionGraphAccessor
id
- id of job vertex to be returnednull
public Map<JobVertexID,ExecutionJobVertex> getAllVertices()
AccessExecutionGraph
getAllVertices
in interface AccessExecutionGraph
getAllVertices
in interface ExecutionGraph
public Iterable<ExecutionJobVertex> getVerticesTopologically()
AccessExecutionGraph
getVerticesTopologically
in interface AccessExecutionGraph
getVerticesTopologically
in interface ExecutionGraph
public Map<IntermediateDataSetID,IntermediateResult> getAllIntermediateResults()
getAllIntermediateResults
in interface ExecutionGraph
public Iterable<ExecutionVertex> getAllExecutionVertices()
AccessExecutionGraph
getAllExecutionVertices
in interface AccessExecutionGraph
getAllExecutionVertices
in interface ExecutionGraph
public EdgeManager getEdgeManager()
getEdgeManager
in interface InternalExecutionGraphAccessor
public ExecutionVertex getExecutionVertexOrThrow(ExecutionVertexID id)
getExecutionVertexOrThrow
in interface InternalExecutionGraphAccessor
public IntermediateResultPartition getResultPartitionOrThrow(IntermediateResultPartitionID id)
getResultPartitionOrThrow
in interface InternalExecutionGraphAccessor
public long getStatusTimestamp(JobStatus status)
AccessExecutionGraph
JobStatus
.getStatusTimestamp
in interface AccessExecutionGraph
getStatusTimestamp
in interface JobStatusProvider
status
- status for which the timestamp should be returnedpublic final BlobWriter getBlobWriter()
getBlobWriter
in interface InternalExecutionGraphAccessor
public Executor getFutureExecutor()
InternalExecutionGraphAccessor
getFutureExecutor
in interface InternalExecutionGraphAccessor
public Map<String,OptionalFailure<Accumulator<?,?>>> aggregateUserAccumulators()
ExecutionGraph
aggregateUserAccumulators
in interface ExecutionGraph
public Map<String,SerializedValue<OptionalFailure<Object>>> getAccumulatorsSerialized()
getAccumulatorsSerialized
in interface AccessExecutionGraph
public StringifiedAccumulatorResult[] getAccumulatorResultsStringified()
getAccumulatorResultsStringified
in interface AccessExecutionGraph
public void setInternalTaskFailuresListener(InternalFailuresListener internalTaskFailuresListener)
setInternalTaskFailuresListener
in interface ExecutionGraph
public void notifyNewlyInitializedJobVertices(List<ExecutionJobVertex> vertices)
ExecutionGraph
notifyNewlyInitializedJobVertices
in interface ExecutionGraph
vertices
- The execution job vertices that are newly initialized.public void attachJobGraph(List<JobVertex> topologicallySorted) throws JobException
attachJobGraph
in interface ExecutionGraph
JobException
public void initializeJobVertex(ExecutionJobVertex ejv, long createTimestamp) throws JobException
ExecutionGraph
initializeJobVertex
in interface ExecutionGraph
ejv
- The execution job vertex that needs to be initialized.createTimestamp
- The timestamp for creating execution vertices, used to initialize the
first Execution with.JobException
public void transitionToRunning()
transitionToRunning
in interface ExecutionGraph
public void cancel()
cancel
in interface ExecutionGraph
@VisibleForTesting protected FutureUtils.ConjunctFuture<Void> cancelVerticesAsync()
public void suspend(Throwable suspensionCause)
ExecutionGraph
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.
suspend
in interface ExecutionGraph
suspensionCause
- Cause of the suspensionpublic void failGlobal(Throwable t)
InternalExecutionGraphAccessor
This global failure is meant to be triggered in cases where the consistency of the execution graph' state cannot be guaranteed any more (for example when catching unexpected exceptions that indicate a bug or an unexpected call race), and where a full restart is the safe way to get consistency back.
failGlobal
in interface InternalExecutionGraphAccessor
t
- The exception that caused the failure.public ArchivedExecutionConfig getArchivedExecutionConfig()
ArchivedExecutionConfig
.getArchivedExecutionConfig
in interface AccessExecutionGraph
public CompletableFuture<JobStatus> getTerminationFuture()
ExecutionGraph
ExecutionGraph
. The termination future is
completed with the terminal JobStatus
once the ExecutionGraph reaches this terminal
state and all Execution
have been terminated.getTerminationFuture
in interface ExecutionGraph
ExecutionGraph
.@VisibleForTesting public JobStatus waitUntilTerminal() throws InterruptedException
waitUntilTerminal
in interface ExecutionGraph
InterruptedException
public boolean transitionState(JobStatus current, JobStatus newState)
transitionState
in interface ExecutionGraph
public void incrementRestarts()
incrementRestarts
in interface ExecutionGraph
public void initFailureCause(Throwable t, long timestamp)
initFailureCause
in interface ExecutionGraph
public void jobVertexFinished()
jobVertexFinished
in interface InternalExecutionGraphAccessor
public void jobVertexUnFinished()
jobVertexUnFinished
in interface InternalExecutionGraphAccessor
public void failJob(Throwable cause, long timestamp)
failJob
in interface ExecutionGraph
public boolean updateState(TaskExecutionStateTransition state)
ExecutionGraph
updateState
in interface ExecutionGraph
state
- The state update.public Map<ExecutionAttemptID,Execution> getRegisteredExecutions()
getRegisteredExecutions
in interface ExecutionGraph
public void registerExecution(Execution exec)
registerExecution
in interface InternalExecutionGraphAccessor
public void deregisterExecution(Execution exec)
deregisterExecution
in interface InternalExecutionGraphAccessor
public void updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
ExecutionGraph
updateAccumulators
in interface ExecutionGraph
accumulatorSnapshot
- The serialized flink and user-defined accumulatorspublic void registerJobStatusListener(JobStatusListener listener)
registerJobStatusListener
in interface ExecutionGraph
public void notifyExecutionChange(Execution execution, ExecutionState previousState, ExecutionState newExecutionState)
notifyExecutionChange
in interface InternalExecutionGraphAccessor
public void notifySchedulerNgAboutInternalTaskFailure(ExecutionAttemptID attemptId, Throwable t, boolean cancelTask, boolean releasePartitions)
notifySchedulerNgAboutInternalTaskFailure
in interface InternalExecutionGraphAccessor
public void deleteBlobs(List<PermanentBlobKey> blobKeys)
deleteBlobs
in interface InternalExecutionGraphAccessor
public ShuffleMaster<?> getShuffleMaster()
getShuffleMaster
in interface InternalExecutionGraphAccessor
public JobMasterPartitionTracker getPartitionTracker()
getPartitionTracker
in interface InternalExecutionGraphAccessor
public ResultPartitionAvailabilityChecker getResultPartitionAvailabilityChecker()
getResultPartitionAvailabilityChecker
in interface ExecutionGraph
public PartitionGroupReleaseStrategy getPartitionGroupReleaseStrategy()
getPartitionGroupReleaseStrategy
in interface InternalExecutionGraphAccessor
public ExecutionDeploymentListener getExecutionDeploymentListener()
getExecutionDeploymentListener
in interface InternalExecutionGraphAccessor
public boolean isDynamic()
isDynamic
in interface InternalExecutionGraphAccessor
public Optional<String> findVertexWithAttempt(ExecutionAttemptID attemptId)
findVertexWithAttempt
in interface ExecutionGraph
public Optional<AccessExecution> findExecution(ExecutionAttemptID attemptId)
findExecution
in interface ExecutionGraph
public ExecutionGraphID getExecutionGraphID()
getExecutionGraphID
in interface InternalExecutionGraphAccessor
public List<ShuffleDescriptor> getClusterPartitionShuffleDescriptors(IntermediateDataSetID intermediateDataSetID)
InternalExecutionGraphAccessor
getClusterPartitionShuffleDescriptors
in interface InternalExecutionGraphAccessor
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.