Class ExecutionJobVertex
- java.lang.Object
-
- org.apache.flink.runtime.executiongraph.ExecutionJobVertex
-
- All Implemented Interfaces:
Archiveable<ArchivedExecutionJobVertex>
,AccessExecutionJobVertex
- Direct Known Subclasses:
SpeculativeExecutionJobVertex
public class ExecutionJobVertex extends Object implements AccessExecutionJobVertex, Archiveable<ArchivedExecutionJobVertex>
AnExecutionJobVertex
is part of theExecutionGraph
, and the peer to theJobVertex
.The
ExecutionJobVertex
corresponds to a parallelized operation. It contains anExecutionVertex
for each parallel instance of that operation.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
ExecutionJobVertex.Factory
Factory to createExecutionJobVertex
.
-
Constructor Summary
Constructors Constructor Description ExecutionJobVertex(InternalExecutionGraphAccessor graph, JobVertex jobVertex, VertexParallelismInformation parallelismInfo, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description ArchivedExecutionJobVertex
archive()
void
cancel()
Cancels all currently running vertex executions.CompletableFuture<Void>
cancelWithFuture()
Cancels all currently running vertex executions.boolean
canRescaleMaxParallelism(int desiredMaxParallelism)
void
connectToPredecessors(Map<IntermediateDataSetID,IntermediateResult> intermediateDataSets)
protected ExecutionVertex
createExecutionVertex(ExecutionJobVertex jobVertex, int subTaskIndex, IntermediateResult[] producedDataSets, Duration timeout, long createTimestamp, int executionHistorySizeLimit, int initialAttemptCount)
protected OperatorCoordinatorHolder
createOperatorCoordinatorHolder(SerializedValue<OperatorCoordinator.Provider> provider, ClassLoader classLoader, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup)
void
fail(Throwable t)
StringifiedAccumulatorResult[]
getAggregatedUserAccumulatorsStringified()
Returns the aggregated user-defined accumulators as strings.static ExecutionState
getAggregateJobVertexState(int[] verticesPerState, int parallelism)
A utility function that computes an "aggregated" state for the vertex.ExecutionState
getAggregateState()
Returns the aggregatedExecutionState
for this job vertex.CoLocationGroup
getCoLocationGroup()
InternalExecutionGraphAccessor
getGraph()
List<IntermediateResult>
getInputs()
JobID
getJobId()
JobVertex
getJobVertex()
JobVertexID
getJobVertexId()
Returns theJobVertexID
for this job vertex.int
getMaxParallelism()
Returns the max parallelism for this job vertex.String
getName()
Returns the name for this job vertex.Collection<OperatorCoordinatorHolder>
getOperatorCoordinators()
List<OperatorIDPair>
getOperatorIDs()
Returns a list containing the ID pairs of all operators contained in this execution job vertex.int
getParallelism()
Returns the parallelism for this job vertex.IntermediateResult[]
getProducedDataSets()
ResourceProfile
getResourceProfile()
Returns the resource profile for this job vertex.SlotSharingGroup
getSlotSharingGroup()
Returns the slot sharing group for this job vertex.List<SourceCoordinator<?,?>>
getSourceCoordinators()
InputSplitAssigner
getSplitAssigner()
TaskInformation
getTaskInformation()
Either<SerializedValue<TaskInformation>,PermanentBlobKey>
getTaskInformationOrBlobKey()
ExecutionVertex[]
getTaskVertices()
Returns all execution vertices for this job vertex.protected void
initialize(int executionHistorySizeLimit, Duration timeout, long createTimestamp, SubtaskAttemptNumberStore initialAttemptCounts)
boolean
isFinished()
boolean
isInitialized()
boolean
isParallelismDecided()
void
setMaxParallelism(int maxParallelism)
void
setParallelism(int parallelism)
CompletableFuture<Void>
suspend()
-
-
-
Constructor Detail
-
ExecutionJobVertex
@VisibleForTesting public ExecutionJobVertex(InternalExecutionGraphAccessor graph, JobVertex jobVertex, VertexParallelismInformation parallelismInfo, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup) throws JobException
- Throws:
JobException
-
-
Method Detail
-
initialize
protected void initialize(int executionHistorySizeLimit, Duration timeout, long createTimestamp, SubtaskAttemptNumberStore initialAttemptCounts) throws JobException
- Throws:
JobException
-
createExecutionVertex
protected ExecutionVertex createExecutionVertex(ExecutionJobVertex jobVertex, int subTaskIndex, IntermediateResult[] producedDataSets, Duration timeout, long createTimestamp, int executionHistorySizeLimit, int initialAttemptCount)
-
createOperatorCoordinatorHolder
protected OperatorCoordinatorHolder createOperatorCoordinatorHolder(SerializedValue<OperatorCoordinator.Provider> provider, ClassLoader classLoader, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup) throws Exception
- Throws:
Exception
-
isInitialized
public boolean isInitialized()
-
isParallelismDecided
public boolean isParallelismDecided()
-
getOperatorIDs
public List<OperatorIDPair> getOperatorIDs()
Returns a list containing the ID pairs of all operators contained in this execution job vertex.- Returns:
- list containing the ID pairs of all contained operators
-
setMaxParallelism
public void setMaxParallelism(int maxParallelism)
-
getGraph
public InternalExecutionGraphAccessor getGraph()
-
setParallelism
public void setParallelism(int parallelism)
-
getJobVertex
public JobVertex getJobVertex()
-
getName
public String getName()
Description copied from interface:AccessExecutionJobVertex
Returns the name for this job vertex.- Specified by:
getName
in interfaceAccessExecutionJobVertex
- Returns:
- name for this job vertex.
-
getParallelism
public int getParallelism()
Description copied from interface:AccessExecutionJobVertex
Returns the parallelism for this job vertex.- Specified by:
getParallelism
in interfaceAccessExecutionJobVertex
- Returns:
- parallelism for this job vertex.
-
getMaxParallelism
public int getMaxParallelism()
Description copied from interface:AccessExecutionJobVertex
Returns the max parallelism for this job vertex.- Specified by:
getMaxParallelism
in interfaceAccessExecutionJobVertex
- Returns:
- max parallelism for this job vertex.
-
getResourceProfile
public ResourceProfile getResourceProfile()
Description copied from interface:AccessExecutionJobVertex
Returns the resource profile for this job vertex.- Specified by:
getResourceProfile
in interfaceAccessExecutionJobVertex
- Returns:
- resource profile for this job vertex.
-
canRescaleMaxParallelism
public boolean canRescaleMaxParallelism(int desiredMaxParallelism)
-
getJobId
public JobID getJobId()
-
getJobVertexId
public JobVertexID getJobVertexId()
Description copied from interface:AccessExecutionJobVertex
Returns theJobVertexID
for this job vertex.- Specified by:
getJobVertexId
in interfaceAccessExecutionJobVertex
- Returns:
- JobVertexID for this job vertex.
-
getTaskVertices
public ExecutionVertex[] getTaskVertices()
Description copied from interface:AccessExecutionJobVertex
Returns all execution vertices for this job vertex.- Specified by:
getTaskVertices
in interfaceAccessExecutionJobVertex
- Returns:
- all execution vertices for this job vertex
-
getProducedDataSets
public IntermediateResult[] getProducedDataSets()
-
getSplitAssigner
public InputSplitAssigner getSplitAssigner()
-
getSlotSharingGroup
public SlotSharingGroup getSlotSharingGroup()
Description copied from interface:AccessExecutionJobVertex
Returns the slot sharing group for this job vertex.- Specified by:
getSlotSharingGroup
in interfaceAccessExecutionJobVertex
- Returns:
- slot sharing group for this job vertex.
-
getCoLocationGroup
@Nullable public CoLocationGroup getCoLocationGroup()
-
getInputs
public List<IntermediateResult> getInputs()
-
getOperatorCoordinators
public Collection<OperatorCoordinatorHolder> getOperatorCoordinators()
-
getSourceCoordinators
public List<SourceCoordinator<?,?>> getSourceCoordinators()
-
getTaskInformationOrBlobKey
public Either<SerializedValue<TaskInformation>,PermanentBlobKey> getTaskInformationOrBlobKey() throws IOException
- Throws:
IOException
-
getTaskInformation
public TaskInformation getTaskInformation()
-
getAggregateState
public ExecutionState getAggregateState()
Description copied from interface:AccessExecutionJobVertex
Returns the aggregatedExecutionState
for this job vertex.- Specified by:
getAggregateState
in interfaceAccessExecutionJobVertex
- Returns:
- aggregated state for this job vertex
-
connectToPredecessors
public void connectToPredecessors(Map<IntermediateDataSetID,IntermediateResult> intermediateDataSets) throws JobException
- Throws:
JobException
-
cancel
public void cancel()
Cancels all currently running vertex executions.
-
cancelWithFuture
public CompletableFuture<Void> cancelWithFuture()
Cancels all currently running vertex executions.- Returns:
- A future that is complete once all tasks have canceled.
-
suspend
public CompletableFuture<Void> suspend()
-
fail
public void fail(Throwable t)
-
isFinished
public boolean isFinished()
-
getAggregatedUserAccumulatorsStringified
public StringifiedAccumulatorResult[] getAggregatedUserAccumulatorsStringified()
Description copied from interface:AccessExecutionJobVertex
Returns the aggregated user-defined accumulators as strings.- Specified by:
getAggregatedUserAccumulatorsStringified
in interfaceAccessExecutionJobVertex
- Returns:
- aggregated user-defined accumulators as strings.
-
archive
public ArchivedExecutionJobVertex archive()
- Specified by:
archive
in interfaceArchiveable<ArchivedExecutionJobVertex>
-
getAggregateJobVertexState
public static ExecutionState getAggregateJobVertexState(int[] verticesPerState, int parallelism)
A utility function that computes an "aggregated" state for the vertex.This state is not used anywhere in the coordination, but can be used for display in dashboards to as a summary for how the particular parallel operation represented by this ExecutionJobVertex is currently behaving.
For example, if at least one parallel task is failed, the aggregate state is failed. If not, and at least one parallel task is cancelling (or cancelled), the aggregate state is cancelling (or cancelled). If all tasks are finished, the aggregate state is finished, and so on.
- Parameters:
verticesPerState
- The number of vertices in each state (indexed by the ordinal of the ExecutionState values).parallelism
- The parallelism of the ExecutionJobVertex- Returns:
- The aggregate state of this ExecutionJobVertex.
-
-