Class DefaultExecutionGraph
- java.lang.Object
-
- org.apache.flink.runtime.executiongraph.DefaultExecutionGraph
-
- All Implemented Interfaces:
AccessExecutionGraph
,ExecutionGraph
,InternalExecutionGraphAccessor
,JobStatusProvider
public class DefaultExecutionGraph extends Object implements ExecutionGraph, InternalExecutionGraphAccessor
Default implementation of theExecutionGraph
.
-
-
Constructor Summary
Constructors Constructor Description DefaultExecutionGraph(JobInformation jobInformation, ScheduledExecutorService futureExecutor, Executor ioExecutor, Duration rpcTimeout, int executionHistorySizeLimit, ClassLoader userClassLoader, BlobWriter blobWriter, PartitionGroupReleaseStrategy.Factory partitionGroupReleaseStrategyFactory, ShuffleMaster<?> shuffleMaster, JobMasterPartitionTracker partitionTracker, ExecutionDeploymentListener executionDeploymentListener, ExecutionStateUpdateListener executionStateUpdateListener, long initializationTimestamp, VertexAttemptNumberStore initialAttemptCounts, VertexParallelismStore vertexParallelismStore, boolean isDynamic, ExecutionJobVertex.Factory executionJobVertexFactory, List<JobStatusHook> jobStatusHooks, MarkPartitionFinishedStrategy markPartitionFinishedStrategy, TaskDeploymentDescriptorFactory taskDeploymentDescriptorFactory, List<JobStatusChangedListener> jobStatusChangedListeners)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description Map<String,OptionalFailure<Accumulator<?,?>>>
aggregateUserAccumulators()
Merges all accumulator results from the tasks previously executed in the Executions.void
attachJobGraph(List<JobVertex> verticesToAttach, JobManagerJobMetricGroup jobManagerJobMetricGroup)
void
cancel()
ResultPartitionID
createResultPartitionId(IntermediateResultPartitionID resultPartitionId)
void
deleteBlobs(List<PermanentBlobKey> blobKeys)
void
deregisterExecution(Execution exec)
void
enableCheckpointing(CheckpointCoordinatorConfiguration chkConfig, List<MasterTriggerRestoreHook<?>> masterHooks, CheckpointIDCounter checkpointIDCounter, CompletedCheckpointStore checkpointStore, StateBackend checkpointStateBackend, CheckpointStorage checkpointStorage, CheckpointStatsTracker statsTracker, CheckpointsCleaner checkpointsCleaner, String changelogStorageName)
void
failGlobal(Throwable t)
Fails the execution graph globally.void
failJob(Throwable cause, long timestamp)
Optional<AccessExecution>
findExecution(ExecutionAttemptID attemptId)
Optional<String>
findVertexWithAttempt(ExecutionAttemptID attemptId)
StringifiedAccumulatorResult[]
getAccumulatorResultsStringified()
Returns the a stringified version of the user-defined accumulators.Map<String,SerializedValue<OptionalFailure<Object>>>
getAccumulatorsSerialized()
Gets a serialized accumulator map.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.ArchivedExecutionConfig
getArchivedExecutionConfig()
Returns the serializableArchivedExecutionConfig
.BlobWriter
getBlobWriter()
Optional<String>
getChangelogStorageName()
Returns the changelog storage name for this ExecutionGraph.CheckpointCoordinator
getCheckpointCoordinator()
CheckpointCoordinatorConfiguration
getCheckpointCoordinatorConfiguration()
Returns theCheckpointCoordinatorConfiguration
ornull
if checkpointing is disabled.CheckpointStatsSnapshot
getCheckpointStatsSnapshot()
Returns a snapshot of the checkpoint statistics ornull
if checkpointing is disabled.Optional<String>
getCheckpointStorageName()
Returns the checkpoint storage name for this ExecutionGraph.List<ShuffleDescriptor>
getClusterPartitionShuffleDescriptors(IntermediateDataSetID intermediateDataSetID)
Get the shuffle descriptors of the cluster partitions ordered by partition number.EdgeManager
getEdgeManager()
ExecutionDeploymentListener
getExecutionDeploymentListener()
ExecutionGraphID
getExecutionGraphID()
ExecutionVertex
getExecutionVertexOrThrow(ExecutionVertexID id)
Throwable
getFailureCause()
ErrorInfo
getFailureInfo()
Returns the exception that caused the job to fail.Executor
getFutureExecutor()
Returns the ExecutionContext associated with this ExecutionGraph.Configuration
getJobConfiguration()
JobID
getJobID()
Returns theJobID
for this execution graph.ComponentMainThreadExecutor
getJobMasterMainThreadExecutor()
String
getJobName()
Returns the job name for the execution graph.JobType
getJobType()
Returns theJobType
for this execution graph.ExecutionJobVertex
getJobVertex(JobVertexID id)
Returns the job vertex for the givenJobVertexID
.JobVertexInputInfo
getJobVertexInputInfo(JobVertexID jobVertexId, IntermediateDataSetID resultId)
Get the input info of a certain input of a certain job vertex.String
getJsonPlan()
Returns the job plan as a JSON string.KvStateLocationRegistry
getKvStateLocationRegistry()
MarkPartitionFinishedStrategy
getMarkPartitionFinishedStrategy()
long
getNumberOfRestarts()
Gets the number of restarts, including full restarts and fine grained restarts.int
getNumFinishedVertices()
PartitionGroupReleaseStrategy
getPartitionGroupReleaseStrategy()
JobMasterPartitionTracker
getPartitionTracker()
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()
ShuffleMaster<?>
getShuffleMaster()
JobStatus
getState()
Returns the currentJobStatus
for this execution graph.Optional<String>
getStateBackendName()
Returns the state backend name for this ExecutionGraph.long
getStatusTimestamp(JobStatus status)
Returns the timestamp for the givenJobStatus
.TaskDeploymentDescriptorFactory
getTaskDeploymentDescriptorFactory()
CompletableFuture<JobStatus>
getTerminationFuture()
Returns the termination future of thisExecutionGraph
.ClassLoader
getUserClassLoader()
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)
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.TernaryBoolean
isChangelogStateBackendEnabled()
Returns whether the state changelog is enabled for this ExecutionGraph.boolean
isDynamic()
boolean
isStoppable()
Returns whether the job for this execution graph is stoppable.void
jobVertexFinished()
Called whenever a job vertex reaches state FINISHED (completed successfully).void
jobVertexUnFinished()
void
notifyExecutionChange(Execution execution, ExecutionState previousState, ExecutionState newExecutionState)
void
notifyNewlyInitializedJobVertices(List<ExecutionJobVertex> vertices)
Notify that some job vertices have been newly initialized, execution graph will try to update scheduling topology.void
notifySchedulerNgAboutInternalTaskFailure(ExecutionAttemptID attemptId, Throwable t, boolean cancelTask, boolean releasePartitions)
void
registerExecution(Execution exec)
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()
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.executiongraph.ExecutionGraph
initializeJobVertex
-
-
-
-
Constructor Detail
-
DefaultExecutionGraph
public DefaultExecutionGraph(JobInformation jobInformation, ScheduledExecutorService futureExecutor, Executor ioExecutor, Duration rpcTimeout, int executionHistorySizeLimit, ClassLoader userClassLoader, BlobWriter blobWriter, PartitionGroupReleaseStrategy.Factory partitionGroupReleaseStrategyFactory, ShuffleMaster<?> shuffleMaster, JobMasterPartitionTracker partitionTracker, ExecutionDeploymentListener executionDeploymentListener, ExecutionStateUpdateListener executionStateUpdateListener, long initializationTimestamp, VertexAttemptNumberStore initialAttemptCounts, VertexParallelismStore vertexParallelismStore, boolean isDynamic, ExecutionJobVertex.Factory executionJobVertexFactory, List<JobStatusHook> jobStatusHooks, MarkPartitionFinishedStrategy markPartitionFinishedStrategy, TaskDeploymentDescriptorFactory taskDeploymentDescriptorFactory, List<JobStatusChangedListener> jobStatusChangedListeners)
-
-
Method Detail
-
start
public void start(@Nonnull ComponentMainThreadExecutor jobMasterMainThreadExecutor)
- Specified by:
start
in interfaceExecutionGraph
-
getSchedulingTopology
public SchedulingTopology getSchedulingTopology()
- Specified by:
getSchedulingTopology
in interfaceExecutionGraph
-
getJobMasterMainThreadExecutor
@Nonnull public ComponentMainThreadExecutor getJobMasterMainThreadExecutor()
- Specified by:
getJobMasterMainThreadExecutor
in interfaceExecutionGraph
- Specified by:
getJobMasterMainThreadExecutor
in interfaceInternalExecutionGraphAccessor
-
isChangelogStateBackendEnabled
public TernaryBoolean isChangelogStateBackendEnabled()
Description copied from interface:AccessExecutionGraph
Returns whether the state changelog is enabled for this ExecutionGraph.- Specified by:
isChangelogStateBackendEnabled
in interfaceAccessExecutionGraph
- Returns:
- true, if state changelog enabled, false otherwise.
-
getStateBackendName
public Optional<String> getStateBackendName()
Description copied from interface:AccessExecutionGraph
Returns the state backend name for this ExecutionGraph.- Specified by:
getStateBackendName
in interfaceAccessExecutionGraph
- Returns:
- The state backend name, or an empty Optional in the case of batch jobs
-
getCheckpointStorageName
public Optional<String> getCheckpointStorageName()
Description copied from interface:AccessExecutionGraph
Returns the checkpoint storage name for this ExecutionGraph.- Specified by:
getCheckpointStorageName
in interfaceAccessExecutionGraph
- Returns:
- The checkpoint storage name, or an empty Optional in the case of batch jobs
-
getChangelogStorageName
public Optional<String> getChangelogStorageName()
Description copied from interface:AccessExecutionGraph
Returns the changelog storage name for this ExecutionGraph.- Specified by:
getChangelogStorageName
in interfaceAccessExecutionGraph
- Returns:
- The changelog storage name, or an empty Optional in the case of batch jobs
-
enableCheckpointing
public void enableCheckpointing(CheckpointCoordinatorConfiguration chkConfig, List<MasterTriggerRestoreHook<?>> masterHooks, CheckpointIDCounter checkpointIDCounter, CompletedCheckpointStore checkpointStore, StateBackend checkpointStateBackend, CheckpointStorage checkpointStorage, CheckpointStatsTracker statsTracker, CheckpointsCleaner checkpointsCleaner, String changelogStorageName)
- Specified by:
enableCheckpointing
in interfaceExecutionGraph
-
getCheckpointCoordinator
@Nullable public CheckpointCoordinator getCheckpointCoordinator()
- Specified by:
getCheckpointCoordinator
in interfaceExecutionGraph
-
getKvStateLocationRegistry
public KvStateLocationRegistry getKvStateLocationRegistry()
- Specified by:
getKvStateLocationRegistry
in interfaceExecutionGraph
-
getCheckpointCoordinatorConfiguration
public CheckpointCoordinatorConfiguration getCheckpointCoordinatorConfiguration()
Description copied from interface:AccessExecutionGraph
Returns theCheckpointCoordinatorConfiguration
ornull
if checkpointing is disabled.- Specified by:
getCheckpointCoordinatorConfiguration
in interfaceAccessExecutionGraph
- Returns:
- JobCheckpointingConfiguration for this execution graph
-
getCheckpointStatsSnapshot
public CheckpointStatsSnapshot getCheckpointStatsSnapshot()
Description copied from interface:AccessExecutionGraph
Returns a snapshot of the checkpoint statistics ornull
if checkpointing is disabled.- Specified by:
getCheckpointStatsSnapshot
in interfaceAccessExecutionGraph
- Returns:
- Snapshot of the checkpoint statistics for this execution graph
-
setJsonPlan
public void setJsonPlan(String jsonPlan)
- Specified by:
setJsonPlan
in interfaceExecutionGraph
-
getJsonPlan
public String getJsonPlan()
Description copied from interface:AccessExecutionGraph
Returns the job plan as a JSON string.- Specified by:
getJsonPlan
in interfaceAccessExecutionGraph
- Returns:
- job plan as a JSON string
-
getJobID
public JobID getJobID()
Description copied from interface:AccessExecutionGraph
Returns theJobID
for this execution graph.- Specified by:
getJobID
in interfaceAccessExecutionGraph
- Specified by:
getJobID
in interfaceInternalExecutionGraphAccessor
- Returns:
- job ID for this execution graph
-
getJobName
public String getJobName()
Description copied from interface:AccessExecutionGraph
Returns the job name for the execution graph.- Specified by:
getJobName
in interfaceAccessExecutionGraph
- Returns:
- job name for this execution graph
-
isStoppable
public boolean isStoppable()
Description copied from interface:AccessExecutionGraph
Returns whether the job for this execution graph is stoppable.- Specified by:
isStoppable
in interfaceAccessExecutionGraph
- Returns:
- true, if all sources tasks are stoppable, false otherwise
-
getJobConfiguration
public Configuration getJobConfiguration()
- Specified by:
getJobConfiguration
in interfaceExecutionGraph
-
getUserClassLoader
public ClassLoader getUserClassLoader()
- Specified by:
getUserClassLoader
in interfaceInternalExecutionGraphAccessor
-
getState
public JobStatus getState()
Description copied from interface:AccessExecutionGraph
Returns the currentJobStatus
for this execution graph.- Specified by:
getState
in interfaceAccessExecutionGraph
- Specified by:
getState
in interfaceJobStatusProvider
- Returns:
- job status for this execution graph
-
getJobType
public JobType getJobType()
Description copied from interface:AccessExecutionGraph
Returns theJobType
for this execution graph.- Specified by:
getJobType
in interfaceAccessExecutionGraph
- Returns:
- job type for this execution graph. It may be null when an exception occurs.
-
getFailureCause
public Throwable getFailureCause()
- Specified by:
getFailureCause
in interfaceExecutionGraph
-
getFailureInfo
public ErrorInfo getFailureInfo()
Description copied from interface:AccessExecutionGraph
Returns the exception that caused the job to fail. This is the first root exception that was not recoverable and triggered job failure.- Specified by:
getFailureInfo
in interfaceAccessExecutionGraph
- Returns:
- failure causing exception, or null
-
getNumberOfRestarts
public long getNumberOfRestarts()
Description copied from interface:ExecutionGraph
Gets the number of restarts, including full restarts and fine grained restarts. If a recovery is currently pending, this recovery is included in the count.- Specified by:
getNumberOfRestarts
in interfaceExecutionGraph
- Returns:
- The number of restarts so far
-
getNumFinishedVertices
public int getNumFinishedVertices()
- Specified by:
getNumFinishedVertices
in interfaceExecutionGraph
-
getJobVertex
public ExecutionJobVertex getJobVertex(JobVertexID id)
Description copied from interface:AccessExecutionGraph
Returns the job vertex for the givenJobVertexID
.- Specified by:
getJobVertex
in interfaceAccessExecutionGraph
- Specified by:
getJobVertex
in interfaceExecutionGraph
- Specified by:
getJobVertex
in interfaceInternalExecutionGraphAccessor
- Parameters:
id
- id of job vertex to be returned- Returns:
- job vertex for the given id, or
null
-
getAllVertices
public Map<JobVertexID,ExecutionJobVertex> getAllVertices()
Description copied from interface:AccessExecutionGraph
Returns a map containing all job vertices for this execution graph.- Specified by:
getAllVertices
in interfaceAccessExecutionGraph
- Specified by:
getAllVertices
in interfaceExecutionGraph
- Returns:
- map containing all job vertices for this execution graph
-
getVerticesTopologically
public Iterable<ExecutionJobVertex> getVerticesTopologically()
Description copied from interface:AccessExecutionGraph
Returns an iterable containing all job vertices for this execution graph in the order they were created.- Specified by:
getVerticesTopologically
in interfaceAccessExecutionGraph
- Specified by:
getVerticesTopologically
in interfaceExecutionGraph
- Returns:
- iterable containing all job vertices for this execution graph in the order they were created
-
getAllIntermediateResults
public Map<IntermediateDataSetID,IntermediateResult> getAllIntermediateResults()
- Specified by:
getAllIntermediateResults
in interfaceExecutionGraph
-
getAllExecutionVertices
public Iterable<ExecutionVertex> getAllExecutionVertices()
Description copied from interface:AccessExecutionGraph
Returns an iterable containing all execution vertices for this execution graph.- Specified by:
getAllExecutionVertices
in interfaceAccessExecutionGraph
- Specified by:
getAllExecutionVertices
in interfaceExecutionGraph
- Returns:
- iterable containing all execution vertices for this execution graph
-
getEdgeManager
public EdgeManager getEdgeManager()
- Specified by:
getEdgeManager
in interfaceInternalExecutionGraphAccessor
-
getExecutionVertexOrThrow
public ExecutionVertex getExecutionVertexOrThrow(ExecutionVertexID id)
- Specified by:
getExecutionVertexOrThrow
in interfaceInternalExecutionGraphAccessor
-
getResultPartitionOrThrow
public IntermediateResultPartition getResultPartitionOrThrow(IntermediateResultPartitionID id)
Description copied from interface:ExecutionGraph
Gets the intermediate result partition by the given partition ID, or throw an exception if the partition is not found.- Specified by:
getResultPartitionOrThrow
in interfaceExecutionGraph
- Specified by:
getResultPartitionOrThrow
in interfaceInternalExecutionGraphAccessor
- Parameters:
id
- of the intermediate result partition- Returns:
- intermediate result partition
-
getStatusTimestamp
public long getStatusTimestamp(JobStatus status)
Description copied from interface:AccessExecutionGraph
Returns the timestamp for the givenJobStatus
.- Specified by:
getStatusTimestamp
in interfaceAccessExecutionGraph
- Specified by:
getStatusTimestamp
in interfaceJobStatusProvider
- Parameters:
status
- status for which the timestamp should be returned- Returns:
- timestamp for the given job status
-
getBlobWriter
public final BlobWriter getBlobWriter()
- Specified by:
getBlobWriter
in interfaceInternalExecutionGraphAccessor
-
getFutureExecutor
public Executor getFutureExecutor()
Description copied from interface:InternalExecutionGraphAccessor
Returns the ExecutionContext associated with this ExecutionGraph.- Specified by:
getFutureExecutor
in interfaceInternalExecutionGraphAccessor
- Returns:
- ExecutionContext associated with this ExecutionGraph
-
aggregateUserAccumulators
public Map<String,OptionalFailure<Accumulator<?,?>>> aggregateUserAccumulators()
Description copied from interface:ExecutionGraph
Merges all accumulator results from the tasks previously executed in the Executions.- Specified by:
aggregateUserAccumulators
in interfaceExecutionGraph
- Returns:
- The accumulator map
-
getAccumulatorsSerialized
public Map<String,SerializedValue<OptionalFailure<Object>>> getAccumulatorsSerialized()
Gets a serialized accumulator map.- Specified by:
getAccumulatorsSerialized
in interfaceAccessExecutionGraph
- Returns:
- The accumulator map with serialized accumulator values.
-
getAccumulatorResultsStringified
public StringifiedAccumulatorResult[] getAccumulatorResultsStringified()
Returns the a stringified version of the user-defined accumulators.- Specified by:
getAccumulatorResultsStringified
in interfaceAccessExecutionGraph
- Returns:
- an Array containing the StringifiedAccumulatorResult objects
-
setInternalTaskFailuresListener
public void setInternalTaskFailuresListener(InternalFailuresListener internalTaskFailuresListener)
- Specified by:
setInternalTaskFailuresListener
in interfaceExecutionGraph
-
notifyNewlyInitializedJobVertices
public void notifyNewlyInitializedJobVertices(List<ExecutionJobVertex> vertices)
Description copied from interface:ExecutionGraph
Notify that some job vertices have been newly initialized, execution graph will try to update scheduling topology.- Specified by:
notifyNewlyInitializedJobVertices
in interfaceExecutionGraph
- Parameters:
vertices
- The execution job vertices that are newly initialized.
-
attachJobGraph
public void attachJobGraph(List<JobVertex> verticesToAttach, JobManagerJobMetricGroup jobManagerJobMetricGroup) throws JobException
- Specified by:
attachJobGraph
in interfaceExecutionGraph
- Throws:
JobException
-
initializeJobVertex
public void initializeJobVertex(ExecutionJobVertex ejv, long createTimestamp, Map<IntermediateDataSetID,JobVertexInputInfo> jobVertexInputInfos) throws JobException
Description copied from interface:ExecutionGraph
Initialize the given execution job vertex, mainly includes creating execution vertices according to the parallelism, and connecting to the predecessors.- Specified by:
initializeJobVertex
in interfaceExecutionGraph
- Parameters:
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.- Throws:
JobException
-
transitionToRunning
public void transitionToRunning()
- Specified by:
transitionToRunning
in interfaceExecutionGraph
-
cancel
public void cancel()
- Specified by:
cancel
in interfaceExecutionGraph
-
suspend
public void suspend(Throwable suspensionCause)
Description copied from interface:ExecutionGraph
Suspends the current 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.- Specified by:
suspend
in interfaceExecutionGraph
- Parameters:
suspensionCause
- Cause of the suspension
-
failGlobal
public void failGlobal(Throwable t)
Description copied from interface:InternalExecutionGraphAccessor
Fails the execution graph globally.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.
- Specified by:
failGlobal
in interfaceInternalExecutionGraphAccessor
- Parameters:
t
- The exception that caused the failure.
-
getArchivedExecutionConfig
public ArchivedExecutionConfig getArchivedExecutionConfig()
Returns the serializableArchivedExecutionConfig
.- Specified by:
getArchivedExecutionConfig
in interfaceAccessExecutionGraph
- Returns:
- ArchivedExecutionConfig which may be null in case of errors
-
getTerminationFuture
public CompletableFuture<JobStatus> getTerminationFuture()
Description copied from interface:ExecutionGraph
Returns the termination future of thisExecutionGraph
. The termination future is completed with the terminalJobStatus
once the ExecutionGraph reaches this terminal state and allExecution
have been terminated.- Specified by:
getTerminationFuture
in interfaceExecutionGraph
- Returns:
- Termination future of this
ExecutionGraph
.
-
waitUntilTerminal
@VisibleForTesting public JobStatus waitUntilTerminal() throws InterruptedException
- Specified by:
waitUntilTerminal
in interfaceExecutionGraph
- Throws:
InterruptedException
-
transitionState
public boolean transitionState(JobStatus current, JobStatus newState)
- Specified by:
transitionState
in interfaceExecutionGraph
-
incrementRestarts
public void incrementRestarts()
- Specified by:
incrementRestarts
in interfaceExecutionGraph
-
initFailureCause
public void initFailureCause(Throwable t, long timestamp)
- Specified by:
initFailureCause
in interfaceExecutionGraph
-
jobVertexFinished
public void jobVertexFinished()
Called whenever a job vertex reaches state FINISHED (completed successfully). Once all job vertices are in the FINISHED state, the program is successfully done.- Specified by:
jobVertexFinished
in interfaceInternalExecutionGraphAccessor
-
jobVertexUnFinished
public void jobVertexUnFinished()
- Specified by:
jobVertexUnFinished
in interfaceInternalExecutionGraphAccessor
-
failJob
public void failJob(Throwable cause, long timestamp)
- Specified by:
failJob
in interfaceExecutionGraph
-
updateState
public boolean updateState(TaskExecutionStateTransition state)
Description copied from interface:ExecutionGraph
Updates the state of one of the ExecutionVertex's Execution attempts. If the new status if "FINISHED", this also updates the accumulators.- Specified by:
updateState
in interfaceExecutionGraph
- Parameters:
state
- The state update.- Returns:
- True, if the task update was properly applied, false, if the execution attempt was not found.
-
createResultPartitionId
@VisibleForTesting public ResultPartitionID createResultPartitionId(IntermediateResultPartitionID resultPartitionId)
-
getRegisteredExecutions
public Map<ExecutionAttemptID,Execution> getRegisteredExecutions()
- Specified by:
getRegisteredExecutions
in interfaceExecutionGraph
-
registerExecution
public void registerExecution(Execution exec)
- Specified by:
registerExecution
in interfaceInternalExecutionGraphAccessor
-
deregisterExecution
public void deregisterExecution(Execution exec)
- Specified by:
deregisterExecution
in interfaceInternalExecutionGraphAccessor
-
updateAccumulators
public void updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
Description copied from interface:ExecutionGraph
Updates the accumulators during the runtime of a job. Final accumulator results are transferred through the UpdateTaskExecutionState message.- Specified by:
updateAccumulators
in interfaceExecutionGraph
- Parameters:
accumulatorSnapshot
- The serialized flink and user-defined accumulators
-
registerJobStatusListener
public void registerJobStatusListener(JobStatusListener listener)
- Specified by:
registerJobStatusListener
in interfaceExecutionGraph
-
notifyExecutionChange
public void notifyExecutionChange(Execution execution, ExecutionState previousState, ExecutionState newExecutionState)
- Specified by:
notifyExecutionChange
in interfaceInternalExecutionGraphAccessor
-
notifySchedulerNgAboutInternalTaskFailure
public void notifySchedulerNgAboutInternalTaskFailure(ExecutionAttemptID attemptId, Throwable t, boolean cancelTask, boolean releasePartitions)
- Specified by:
notifySchedulerNgAboutInternalTaskFailure
in interfaceInternalExecutionGraphAccessor
-
deleteBlobs
public void deleteBlobs(List<PermanentBlobKey> blobKeys)
- Specified by:
deleteBlobs
in interfaceInternalExecutionGraphAccessor
-
getShuffleMaster
public ShuffleMaster<?> getShuffleMaster()
- Specified by:
getShuffleMaster
in interfaceInternalExecutionGraphAccessor
-
getPartitionTracker
public JobMasterPartitionTracker getPartitionTracker()
- Specified by:
getPartitionTracker
in interfaceInternalExecutionGraphAccessor
-
getResultPartitionAvailabilityChecker
public ResultPartitionAvailabilityChecker getResultPartitionAvailabilityChecker()
- Specified by:
getResultPartitionAvailabilityChecker
in interfaceExecutionGraph
-
getPartitionGroupReleaseStrategy
public PartitionGroupReleaseStrategy getPartitionGroupReleaseStrategy()
- Specified by:
getPartitionGroupReleaseStrategy
in interfaceInternalExecutionGraphAccessor
-
getExecutionDeploymentListener
public ExecutionDeploymentListener getExecutionDeploymentListener()
- Specified by:
getExecutionDeploymentListener
in interfaceInternalExecutionGraphAccessor
-
isDynamic
public boolean isDynamic()
- Specified by:
isDynamic
in interfaceInternalExecutionGraphAccessor
-
findVertexWithAttempt
public Optional<String> findVertexWithAttempt(ExecutionAttemptID attemptId)
- Specified by:
findVertexWithAttempt
in interfaceExecutionGraph
-
findExecution
public Optional<AccessExecution> findExecution(ExecutionAttemptID attemptId)
- Specified by:
findExecution
in interfaceExecutionGraph
-
getExecutionGraphID
public ExecutionGraphID getExecutionGraphID()
- Specified by:
getExecutionGraphID
in interfaceInternalExecutionGraphAccessor
-
getClusterPartitionShuffleDescriptors
public List<ShuffleDescriptor> getClusterPartitionShuffleDescriptors(IntermediateDataSetID intermediateDataSetID)
Description copied from interface:InternalExecutionGraphAccessor
Get the shuffle descriptors of the cluster partitions ordered by partition number.- Specified by:
getClusterPartitionShuffleDescriptors
in interfaceInternalExecutionGraphAccessor
-
getMarkPartitionFinishedStrategy
public MarkPartitionFinishedStrategy getMarkPartitionFinishedStrategy()
- Specified by:
getMarkPartitionFinishedStrategy
in interfaceInternalExecutionGraphAccessor
-
getJobVertexInputInfo
public JobVertexInputInfo getJobVertexInputInfo(JobVertexID jobVertexId, IntermediateDataSetID resultId)
Description copied from interface:InternalExecutionGraphAccessor
Get the input info of a certain input of a certain job vertex.- Specified by:
getJobVertexInputInfo
in interfaceInternalExecutionGraphAccessor
- Parameters:
jobVertexId
- the job vertex idresultId
- the input(intermediate result) id- Returns:
- the input info
-
getTaskDeploymentDescriptorFactory
public TaskDeploymentDescriptorFactory getTaskDeploymentDescriptorFactory()
- Specified by:
getTaskDeploymentDescriptorFactory
in interfaceInternalExecutionGraphAccessor
-
-