public class ExecutionJobVertex extends Object implements AccessExecutionJobVertex, Archiveable<ArchivedExecutionJobVertex>
ExecutionJobVertex
is part of the ExecutionGraph
, and the peer to the JobVertex
.
The ExecutionJobVertex
corresponds to a parallelized operation. It contains an ExecutionVertex
for each parallel instance of that operation.
Modifier and Type | Class and Description |
---|---|
static class |
ExecutionJobVertex.Factory
Factory to create
ExecutionJobVertex . |
Constructor and Description |
---|
ExecutionJobVertex(InternalExecutionGraphAccessor graph,
JobVertex jobVertex,
VertexParallelismInformation parallelismInfo,
CoordinatorStore coordinatorStore,
JobManagerJobMetricGroup jobManagerJobMetricGroup) |
Modifier and Type | Method and 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,
Time 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 aggregated
ExecutionState for this job vertex. |
CoLocationGroup |
getCoLocationGroup() |
InternalExecutionGraphAccessor |
getGraph() |
List<IntermediateResult> |
getInputs() |
JobID |
getJobId() |
JobVertex |
getJobVertex() |
JobVertexID |
getJobVertexId()
Returns the
JobVertexID 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,
Time timeout,
long createTimestamp,
SubtaskAttemptNumberStore initialAttemptCounts) |
boolean |
isFinished() |
boolean |
isInitialized() |
boolean |
isParallelismDecided() |
void |
setMaxParallelism(int maxParallelism) |
void |
setParallelism(int parallelism) |
CompletableFuture<Void> |
suspend() |
@VisibleForTesting public ExecutionJobVertex(InternalExecutionGraphAccessor graph, JobVertex jobVertex, VertexParallelismInformation parallelismInfo, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup) throws JobException
JobException
protected void initialize(int executionHistorySizeLimit, Time timeout, long createTimestamp, SubtaskAttemptNumberStore initialAttemptCounts) throws JobException
JobException
protected ExecutionVertex createExecutionVertex(ExecutionJobVertex jobVertex, int subTaskIndex, IntermediateResult[] producedDataSets, Time timeout, long createTimestamp, int executionHistorySizeLimit, int initialAttemptCount)
protected OperatorCoordinatorHolder createOperatorCoordinatorHolder(SerializedValue<OperatorCoordinator.Provider> provider, ClassLoader classLoader, CoordinatorStore coordinatorStore, JobManagerJobMetricGroup jobManagerJobMetricGroup) throws Exception
Exception
public boolean isInitialized()
public boolean isParallelismDecided()
public List<OperatorIDPair> getOperatorIDs()
public void setMaxParallelism(int maxParallelism)
public InternalExecutionGraphAccessor getGraph()
public void setParallelism(int parallelism)
public JobVertex getJobVertex()
public String getName()
AccessExecutionJobVertex
getName
in interface AccessExecutionJobVertex
public int getParallelism()
AccessExecutionJobVertex
getParallelism
in interface AccessExecutionJobVertex
public int getMaxParallelism()
AccessExecutionJobVertex
getMaxParallelism
in interface AccessExecutionJobVertex
public ResourceProfile getResourceProfile()
AccessExecutionJobVertex
getResourceProfile
in interface AccessExecutionJobVertex
public boolean canRescaleMaxParallelism(int desiredMaxParallelism)
public JobID getJobId()
public JobVertexID getJobVertexId()
AccessExecutionJobVertex
JobVertexID
for this job vertex.getJobVertexId
in interface AccessExecutionJobVertex
public ExecutionVertex[] getTaskVertices()
AccessExecutionJobVertex
getTaskVertices
in interface AccessExecutionJobVertex
public IntermediateResult[] getProducedDataSets()
public InputSplitAssigner getSplitAssigner()
public SlotSharingGroup getSlotSharingGroup()
AccessExecutionJobVertex
getSlotSharingGroup
in interface AccessExecutionJobVertex
@Nullable public CoLocationGroup getCoLocationGroup()
public List<IntermediateResult> getInputs()
public Collection<OperatorCoordinatorHolder> getOperatorCoordinators()
public List<SourceCoordinator<?,?>> getSourceCoordinators()
public Either<SerializedValue<TaskInformation>,PermanentBlobKey> getTaskInformationOrBlobKey() throws IOException
IOException
public TaskInformation getTaskInformation()
public ExecutionState getAggregateState()
AccessExecutionJobVertex
ExecutionState
for this job vertex.getAggregateState
in interface AccessExecutionJobVertex
public void connectToPredecessors(Map<IntermediateDataSetID,IntermediateResult> intermediateDataSets) throws JobException
JobException
public void cancel()
public CompletableFuture<Void> cancelWithFuture()
public CompletableFuture<Void> suspend()
public void fail(Throwable t)
public boolean isFinished()
public StringifiedAccumulatorResult[] getAggregatedUserAccumulatorsStringified()
AccessExecutionJobVertex
getAggregatedUserAccumulatorsStringified
in interface AccessExecutionJobVertex
public ArchivedExecutionJobVertex archive()
archive
in interface Archiveable<ArchivedExecutionJobVertex>
public static ExecutionState getAggregateJobVertexState(int[] verticesPerState, int parallelism)
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.
verticesPerState
- The number of vertices in each state (indexed by the ordinal of the
ExecutionState values).parallelism
- The parallelism of the ExecutionJobVertexCopyright © 2014–2024 The Apache Software Foundation. All rights reserved.