Class ArchivedExecutionGraph
- java.lang.Object
-
- org.apache.flink.runtime.executiongraph.ArchivedExecutionGraph
-
- All Implemented Interfaces:
Serializable
,AccessExecutionGraph
,JobStatusProvider
public class ArchivedExecutionGraph extends Object implements AccessExecutionGraph, Serializable
An archived execution graph represents a serializable form of anExecutionGraph
.- See Also:
- Serialized Form
-
-
Constructor Summary
Constructors Constructor Description ArchivedExecutionGraph(JobID jobID, String jobName, Map<JobVertexID,ArchivedExecutionJobVertex> tasks, List<ArchivedExecutionJobVertex> verticesInCreationOrder, long[] stateTimestamps, JobStatus state, JobType jobType, ErrorInfo failureCause, String jsonPlan, StringifiedAccumulatorResult[] archivedUserAccumulators, Map<String,SerializedValue<OptionalFailure<Object>>> serializedUserAccumulators, ArchivedExecutionConfig executionConfig, boolean isStoppable, CheckpointCoordinatorConfiguration jobCheckpointingConfiguration, CheckpointStatsSnapshot checkpointStatsSnapshot, String stateBackendName, String checkpointStorageName, TernaryBoolean stateChangelogEnabled, String changelogStorageName)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description static ArchivedExecutionGraph
createFrom(ExecutionGraph executionGraph)
Create aArchivedExecutionGraph
from the givenExecutionGraph
.static ArchivedExecutionGraph
createFrom(ExecutionGraph executionGraph, JobStatus statusOverride)
Create aArchivedExecutionGraph
from the givenExecutionGraph
.static ArchivedExecutionGraph
createSparseArchivedExecutionGraph(JobID jobId, String jobName, JobStatus jobStatus, JobType jobType, Throwable throwable, JobCheckpointingSettings checkpointingSettings, long initializationTimestamp)
Create a sparse ArchivedExecutionGraph for a job.static ArchivedExecutionGraph
createSparseArchivedExecutionGraphWithJobVertices(JobID jobId, String jobName, JobStatus jobStatus, JobType jobType, Throwable throwable, JobCheckpointingSettings checkpointingSettings, long initializationTimestamp, Iterable<JobVertex> jobVertices, VertexParallelismStore initialParallelismStore)
StringifiedAccumulatorResult[]
getAccumulatorResultsStringified()
Returns the aggregated user-defined accumulators as strings.Map<String,SerializedValue<OptionalFailure<Object>>>
getAccumulatorsSerialized()
Returns a map containing the serialized values of user-defined accumulators.Iterable<ArchivedExecutionVertex>
getAllExecutionVertices()
Returns an iterable containing all execution vertices for this execution graph.Map<JobVertexID,AccessExecutionJobVertex>
getAllVertices()
Returns a map containing all job vertices for this execution graph.ArchivedExecutionConfig
getArchivedExecutionConfig()
Returns theArchivedExecutionConfig
for this execution graph.Optional<String>
getChangelogStorageName()
Returns the changelog storage name for this ExecutionGraph.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.ErrorInfo
getFailureInfo()
Returns the exception that caused the job to fail.JobID
getJobID()
Returns theJobID
for this execution graph.String
getJobName()
Returns the job name for the execution graph.JobType
getJobType()
Returns theJobType
for this execution graph.ArchivedExecutionJobVertex
getJobVertex(JobVertexID id)
Returns the job vertex for the givenJobVertexID
.String
getJsonPlan()
Returns the job plan as a JSON string.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
.Iterable<ArchivedExecutionJobVertex>
getVerticesTopologically()
Returns an iterable containing all job vertices for this execution graph in the order they were created.TernaryBoolean
isChangelogStateBackendEnabled()
Returns whether the state changelog is enabled for this ExecutionGraph.boolean
isStoppable()
Returns whether the job for this execution graph is stoppable.
-
-
-
Constructor Detail
-
ArchivedExecutionGraph
public ArchivedExecutionGraph(JobID jobID, String jobName, Map<JobVertexID,ArchivedExecutionJobVertex> tasks, List<ArchivedExecutionJobVertex> verticesInCreationOrder, long[] stateTimestamps, JobStatus state, @Nullable JobType jobType, @Nullable ErrorInfo failureCause, String jsonPlan, StringifiedAccumulatorResult[] archivedUserAccumulators, Map<String,SerializedValue<OptionalFailure<Object>>> serializedUserAccumulators, ArchivedExecutionConfig executionConfig, boolean isStoppable, @Nullable CheckpointCoordinatorConfiguration jobCheckpointingConfiguration, @Nullable CheckpointStatsSnapshot checkpointStatsSnapshot, @Nullable String stateBackendName, @Nullable String checkpointStorageName, @Nullable TernaryBoolean stateChangelogEnabled, @Nullable String changelogStorageName)
-
-
Method Detail
-
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
- 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
-
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.
-
getFailureInfo
@Nullable 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
-
getJobVertex
public ArchivedExecutionJobVertex getJobVertex(JobVertexID id)
Description copied from interface:AccessExecutionGraph
Returns the job vertex for the givenJobVertexID
.- Specified by:
getJobVertex
in interfaceAccessExecutionGraph
- Parameters:
id
- id of job vertex to be returned- Returns:
- job vertex for the given id, or
null
-
getAllVertices
public Map<JobVertexID,AccessExecutionJobVertex> getAllVertices()
Description copied from interface:AccessExecutionGraph
Returns a map containing all job vertices for this execution graph.- Specified by:
getAllVertices
in interfaceAccessExecutionGraph
- Returns:
- map containing all job vertices for this execution graph
-
getVerticesTopologically
public Iterable<ArchivedExecutionJobVertex> 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
- Returns:
- iterable containing all job vertices for this execution graph in the order they were created
-
getAllExecutionVertices
public Iterable<ArchivedExecutionVertex> getAllExecutionVertices()
Description copied from interface:AccessExecutionGraph
Returns an iterable containing all execution vertices for this execution graph.- Specified by:
getAllExecutionVertices
in interfaceAccessExecutionGraph
- Returns:
- iterable containing all execution vertices for this execution graph
-
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
-
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
-
getArchivedExecutionConfig
public ArchivedExecutionConfig getArchivedExecutionConfig()
Description copied from interface:AccessExecutionGraph
Returns theArchivedExecutionConfig
for this execution graph.- Specified by:
getArchivedExecutionConfig
in interfaceAccessExecutionGraph
- Returns:
- execution config summary for this execution graph, or null in case of errors
-
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
-
getAccumulatorResultsStringified
public StringifiedAccumulatorResult[] getAccumulatorResultsStringified()
Description copied from interface:AccessExecutionGraph
Returns the aggregated user-defined accumulators as strings.- Specified by:
getAccumulatorResultsStringified
in interfaceAccessExecutionGraph
- Returns:
- aggregated user-defined accumulators as strings.
-
getAccumulatorsSerialized
public Map<String,SerializedValue<OptionalFailure<Object>>> getAccumulatorsSerialized()
Description copied from interface:AccessExecutionGraph
Returns a map containing the serialized values of user-defined accumulators.- Specified by:
getAccumulatorsSerialized
in interfaceAccessExecutionGraph
- Returns:
- map containing serialized values of user-defined accumulators
-
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
-
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.
-
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
-
createFrom
public static ArchivedExecutionGraph createFrom(ExecutionGraph executionGraph)
Create aArchivedExecutionGraph
from the givenExecutionGraph
.- Parameters:
executionGraph
- to create the ArchivedExecutionGraph from- Returns:
- ArchivedExecutionGraph created from the given ExecutionGraph
-
createFrom
public static ArchivedExecutionGraph createFrom(ExecutionGraph executionGraph, @Nullable JobStatus statusOverride)
Create aArchivedExecutionGraph
from the givenExecutionGraph
.- Parameters:
executionGraph
- to create the ArchivedExecutionGraph fromstatusOverride
- optionally overrides the JobStatus of the ExecutionGraph with a non-globally-terminal state and clears timestamps of globally-terminal states- Returns:
- ArchivedExecutionGraph created from the given ExecutionGraph
-
createSparseArchivedExecutionGraph
public static ArchivedExecutionGraph createSparseArchivedExecutionGraph(JobID jobId, String jobName, JobStatus jobStatus, @Nullable JobType jobType, @Nullable Throwable throwable, @Nullable JobCheckpointingSettings checkpointingSettings, long initializationTimestamp)
Create a sparse ArchivedExecutionGraph for a job. Most fields will be empty, only job status and error-related fields are set.
-
createSparseArchivedExecutionGraphWithJobVertices
public static ArchivedExecutionGraph createSparseArchivedExecutionGraphWithJobVertices(JobID jobId, String jobName, JobStatus jobStatus, JobType jobType, @Nullable Throwable throwable, @Nullable JobCheckpointingSettings checkpointingSettings, long initializationTimestamp, Iterable<JobVertex> jobVertices, VertexParallelismStore initialParallelismStore)
-
-