Class SpeculativeExecutionVertex
- java.lang.Object
-
- org.apache.flink.runtime.executiongraph.ExecutionVertex
-
- org.apache.flink.runtime.executiongraph.SpeculativeExecutionVertex
-
- All Implemented Interfaces:
Archiveable<ArchivedExecutionVertex>
,AccessExecutionVertex
public class SpeculativeExecutionVertex extends ExecutionVertex
The ExecutionVertex which supports speculative execution.
-
-
Field Summary
-
Fields inherited from class org.apache.flink.runtime.executiongraph.ExecutionVertex
NUM_BYTES_UNKNOWN
-
-
Constructor Summary
Constructors Constructor Description SpeculativeExecutionVertex(ExecutionJobVertex jobVertex, int subTaskIndex, IntermediateResult[] producedDataSets, Duration timeout, long createTimestamp, int executionHistorySizeLimit, int initialAttemptCount)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description ArchivedExecutionVertex
archive()
void
archiveFailedExecution(ExecutionAttemptID executionAttemptId)
Remove execution from currentExecutions if it is failed.CompletableFuture<?>
cancel()
Cancels this ExecutionVertex.Execution
createNewSpeculativeExecution(long timestamp)
void
deploy()
void
deployToSlot(LogicalSlot slot)
void
fail(Throwable t)
Execution
getCurrentExecution(int attemptNumber)
Execution
getCurrentExecutionAttempt()
Returns the current execution for this execution vertex.Collection<Execution>
getCurrentExecutions()
Returns the current executions for this execution vertex.Optional<InputSplit>
getNextInputSplit(String host, int attemptNumber)
Execution
getPartitionProducer()
Optional<TaskManagerLocation>
getPreferredLocationBasedOnState()
Gets the preferred location to execute the current task execution attempt, based on the state that the execution attempt will resume.boolean
isOriginalAttempt(int attemptNumber)
Returns whether the given attempt is the original execution attempt of the execution vertex, i.e. it is created along with the creation of resetting of the execution vertex.boolean
isSupportsConcurrentExecutionAttempts()
void
markFailed(Throwable t)
This method marks the task as failed, but will make no attempt to remove task execution from the task manager.void
resetForNewExecution()
Archives the current Execution and creates a new Execution for this vertex.CompletableFuture<?>
suspend()
void
tryAssignResource(LogicalSlot slot)
-
Methods inherited from class org.apache.flink.runtime.executiongraph.ExecutionVertex
addConsumedPartitionGroup, findLastAllocation, findLastLocation, finishPartitionsIfNeeded, getAllConsumedPartitionGroups, getConsumedPartitionGroup, getCurrentAssignedResource, getCurrentAssignedResourceLocation, getCurrentTaskManagerLocationFuture, getExecutionGraphAccessor, getExecutionHistory, getExecutionState, getExecutionVertexInputInfo, getFailureInfo, getID, getInputBytes, getJobId, getJobVertex, getJobvertexId, getMaxParallelism, getNumberOfInputs, getParallelSubtaskIndex, getProducedPartitions, getResourceProfile, getStateTimestamp, getTaskName, getTaskNameWithSubtaskIndex, getTotalNumberOfParallelSubtasks, setInputBytes, toString
-
-
-
-
Constructor Detail
-
SpeculativeExecutionVertex
public SpeculativeExecutionVertex(ExecutionJobVertex jobVertex, int subTaskIndex, IntermediateResult[] producedDataSets, Duration timeout, long createTimestamp, int executionHistorySizeLimit, int initialAttemptCount)
-
-
Method Detail
-
isSupportsConcurrentExecutionAttempts
public boolean isSupportsConcurrentExecutionAttempts()
-
createNewSpeculativeExecution
public Execution createNewSpeculativeExecution(long timestamp)
-
isOriginalAttempt
public boolean isOriginalAttempt(int attemptNumber)
Returns whether the given attempt is the original execution attempt of the execution vertex, i.e. it is created along with the creation of resetting of the execution vertex.
-
getCurrentExecutions
public Collection<Execution> getCurrentExecutions()
Description copied from interface:AccessExecutionVertex
Returns the current executions for this execution vertex. The returned collection must contain the current execution attempt.- Specified by:
getCurrentExecutions
in interfaceAccessExecutionVertex
- Overrides:
getCurrentExecutions
in classExecutionVertex
- Returns:
- current executions
-
getCurrentExecution
public Execution getCurrentExecution(int attemptNumber)
- Overrides:
getCurrentExecution
in classExecutionVertex
-
getPartitionProducer
public Execution getPartitionProducer()
- Overrides:
getPartitionProducer
in classExecutionVertex
-
cancel
public CompletableFuture<?> cancel()
Description copied from class:ExecutionVertex
Cancels this ExecutionVertex.- Overrides:
cancel
in classExecutionVertex
- Returns:
- A future that completes once the execution has reached its final state.
-
suspend
public CompletableFuture<?> suspend()
- Overrides:
suspend
in classExecutionVertex
-
fail
public void fail(Throwable t)
- Overrides:
fail
in classExecutionVertex
-
markFailed
public void markFailed(Throwable t)
Description copied from class:ExecutionVertex
This method marks the task as failed, but will make no attempt to remove task execution from the task manager. It is intended for cases where the task is known not to be deployed yet.- Overrides:
markFailed
in classExecutionVertex
- Parameters:
t
- The exception that caused the task to fail.
-
resetForNewExecution
public void resetForNewExecution()
Description copied from class:ExecutionVertex
Archives the current Execution and creates a new Execution for this vertex.- Overrides:
resetForNewExecution
in classExecutionVertex
-
archiveFailedExecution
public void archiveFailedExecution(ExecutionAttemptID executionAttemptId)
Remove execution from currentExecutions if it is failed. It is needed to make room for possible future speculative executions.- Parameters:
executionAttemptId
- attemptID of the execution to be removed
-
getCurrentExecutionAttempt
public Execution getCurrentExecutionAttempt()
Description copied from interface:AccessExecutionVertex
Returns the current execution for this execution vertex.- Specified by:
getCurrentExecutionAttempt
in interfaceAccessExecutionVertex
- Overrides:
getCurrentExecutionAttempt
in classExecutionVertex
- Returns:
- current execution
-
getNextInputSplit
public Optional<InputSplit> getNextInputSplit(String host, int attemptNumber)
- Overrides:
getNextInputSplit
in classExecutionVertex
-
archive
public ArchivedExecutionVertex archive()
- Specified by:
archive
in interfaceArchiveable<ArchivedExecutionVertex>
- Overrides:
archive
in classExecutionVertex
-
tryAssignResource
public void tryAssignResource(LogicalSlot slot)
- Overrides:
tryAssignResource
in classExecutionVertex
-
deploy
public void deploy()
- Overrides:
deploy
in classExecutionVertex
-
deployToSlot
public void deployToSlot(LogicalSlot slot)
- Overrides:
deployToSlot
in classExecutionVertex
-
getPreferredLocationBasedOnState
public Optional<TaskManagerLocation> getPreferredLocationBasedOnState()
Description copied from class:ExecutionVertex
Gets the preferred location to execute the current task execution attempt, based on the state that the execution attempt will resume.- Overrides:
getPreferredLocationBasedOnState
in classExecutionVertex
-
-