public class Execution extends Object implements Serializable
ExecutionVertex
can be executed multiple times (for recovery,
or other re-computation), this class tracks the state of a single execution of that vertex and the resources.
NOTE ABOUT THE DESIGN RATIONAL:
In several points of the code, we need to deal with possible concurrent state changes and actions.
For example, while the call to deploy a task (send it to the TaskManager) happens, the task gets cancelled.
We could lock the entire portion of the code (decision to deploy, deploy, set state to running) such that
it is guaranteed that any "cancel command" will only pick up after deployment is done and that the "cancel
command" call will never overtake the deploying call.
This blocks the threads big time, because the remote calls may take long. Depending of their locking behavior, it
may even result in distributed deadlocks (unless carefully avoided). We therefore use atomic state updates and
occasional double-checking to ensure that the state after a completed call is as expected, and trigger correcting
actions if it is not. Many actions are also idempotent (like canceling).Constructor and Description |
---|
Execution(scala.concurrent.ExecutionContext executionContext,
ExecutionVertex vertex,
int attemptNumber,
long startTimestamp,
scala.concurrent.duration.FiniteDuration timeout) |
Modifier and Type | Method and Description |
---|---|
void |
cancel() |
void |
deployToSlot(SimpleSlot slot) |
void |
fail(Throwable t)
This method fails the vertex due to an external condition.
|
SimpleSlot |
getAssignedResource() |
InstanceConnectionInfo |
getAssignedResourceLocation() |
ExecutionAttemptID |
getAttemptId() |
int |
getAttemptNumber() |
Throwable |
getFailureCause() |
Map<AccumulatorRegistry.Metric,Accumulator<?,?>> |
getFlinkAccumulators() |
ExecutionState |
getState() |
long |
getStateTimestamp(ExecutionState state) |
long[] |
getStateTimestamps() |
Map<String,Accumulator<?,?>> |
getUserAccumulators() |
StringifiedAccumulatorResult[] |
getUserAccumulatorsStringified() |
ExecutionVertex |
getVertex() |
String |
getVertexWithAttempt() |
boolean |
isFinished() |
void |
prepareForArchiving()
This method cleans fields that are irrelevant for the archived execution attempt.
|
boolean |
scheduleForExecution(Scheduler scheduler,
boolean queued)
NOTE: This method only throws exceptions if it is in an illegal state to be scheduled, or if the tasks needs
to be scheduled immediately and no resource is available.
|
void |
setAccumulators(Map<AccumulatorRegistry.Metric,Accumulator<?,?>> flinkAccumulators,
Map<String,Accumulator<?,?>> userAccumulators)
Update accumulators (discarded when the Execution has already been terminated).
|
void |
setInitialState(SerializedValue<StateHandle<?>> initialState,
long recoveryTimestamp) |
void |
stop()
Sends stop RPC call.
|
String |
toString() |
public Execution(scala.concurrent.ExecutionContext executionContext, ExecutionVertex vertex, int attemptNumber, long startTimestamp, scala.concurrent.duration.FiniteDuration timeout)
public ExecutionVertex getVertex()
public ExecutionAttemptID getAttemptId()
public int getAttemptNumber()
public ExecutionState getState()
public SimpleSlot getAssignedResource()
public InstanceConnectionInfo getAssignedResourceLocation()
public Throwable getFailureCause()
public long[] getStateTimestamps()
public long getStateTimestamp(ExecutionState state)
public boolean isFinished()
public void prepareForArchiving()
public void setInitialState(SerializedValue<StateHandle<?>> initialState, long recoveryTimestamp)
public boolean scheduleForExecution(Scheduler scheduler, boolean queued) throws NoResourceAvailableException
scheduler
- The scheduler to use to schedule this execution attempt.queued
- Flag to indicate whether the scheduler may queue this task if it cannot
immediately deploy it.IllegalStateException
- Thrown, if the vertex is not in CREATED state, which is the only state that permits scheduling.NoResourceAvailableException
- Thrown is no queued scheduling is allowed and no resources are currently available.public void deployToSlot(SimpleSlot slot) throws JobException
JobException
public void stop()
public void cancel()
public void fail(Throwable t)
t
- The exception that caused the task to fail.public String getVertexWithAttempt()
public void setAccumulators(Map<AccumulatorRegistry.Metric,Accumulator<?,?>> flinkAccumulators, Map<String,Accumulator<?,?>> userAccumulators)
flinkAccumulators
- the flink internal accumulatorsuserAccumulators
- the user accumulatorspublic Map<String,Accumulator<?,?>> getUserAccumulators()
public StringifiedAccumulatorResult[] getUserAccumulatorsStringified()
public Map<AccumulatorRegistry.Metric,Accumulator<?,?>> getFlinkAccumulators()
Copyright © 2014–2017 The Apache Software Foundation. All rights reserved.