@InterfaceAudience.Private
@InterfaceStability.Stable
public interface TaskUmbilicalProtocol
extends org.apache.hadoop.ipc.VersionedProtocol
Modifier and Type | Field and Description |
---|---|
static long |
versionID
Changed the version to 2, since we have a new method getMapOutputs
Changed version to 3 to have progress() return a boolean
Changed the version to 4, since we have replaced
TaskUmbilicalProtocol.progress(String, float, String,
org.apache.hadoop.mapred.TaskStatus.Phase, Counters)
with statusUpdate(String, TaskStatus)
Version 5 changed counters representation for HADOOP-2248
Version 6 changes the TaskStatus representation for HADOOP-2208
Version 7 changes the done api (via HADOOP-3140).
|
Modifier and Type | Method and Description |
---|---|
boolean |
canCommit(TaskAttemptID taskid)
Polling to know whether the task can go-ahead with commit
|
void |
commitPending(TaskAttemptID taskId,
TaskStatus taskStatus)
Report that the task is complete, but its commit is pending.
|
void |
done(TaskAttemptID taskid)
Report that the task is successfully completed.
|
void |
fatalError(TaskAttemptID taskId,
String message,
boolean fastFail)
Report that the task encounted a fatal error.
|
void |
fsError(TaskAttemptID taskId,
String message)
Report that the task encounted a local filesystem error.
|
TaskCheckpointID |
getCheckpointID(TaskID taskID)
Return the latest CheckpointID for the given TaskID.
|
MapTaskCompletionEventsUpdate |
getMapCompletionEvents(JobID jobId,
int fromIndex,
int maxLocs,
TaskAttemptID id)
Called by a reduce task to get the map output locations for finished maps.
|
JvmTask |
getTask(org.apache.hadoop.mapred.JvmContext context)
Called when a child task process starts, to get its task.
|
void |
preempted(TaskAttemptID taskId,
TaskStatus taskStatus)
Report to the AM that the task has been succesfully preempted.
|
void |
reportDiagnosticInfo(TaskAttemptID taskid,
String trace)
Report error messages back to parent.
|
void |
reportNextRecordRange(TaskAttemptID taskid,
org.apache.hadoop.mapred.SortedRanges.Range range)
Report the record range which is going to process next by the Task.
|
void |
setCheckpointID(TaskID tid,
TaskCheckpointID cid)
Send a CheckpointID for a given TaskID to be stored in the AM,
to later restart a task from this checkpoint.
|
void |
shuffleError(TaskAttemptID taskId,
String message)
Report that a reduce-task couldn't shuffle map-outputs.
|
AMFeedback |
statusUpdate(TaskAttemptID taskId,
TaskStatus taskStatus)
Report child's progress to parent.
|
static final long versionID
JvmTask getTask(org.apache.hadoop.mapred.JvmContext context) throws IOException
context
- the JvmContext of the JVM w.r.t the TaskTracker that
launched itIOException
AMFeedback statusUpdate(TaskAttemptID taskId, TaskStatus taskStatus) throws IOException, InterruptedException
taskId
- task-id of the childtaskStatus
- status of the childIOException
InterruptedException
void reportDiagnosticInfo(TaskAttemptID taskid, String trace) throws IOException
taskid
- the id of the task involvedtrace
- the text to reportIOException
void reportNextRecordRange(TaskAttemptID taskid, org.apache.hadoop.mapred.SortedRanges.Range range) throws IOException
taskid
- the id of the task involvedrange
- the range of record sequence nosIOException
void done(TaskAttemptID taskid) throws IOException
taskid
- task's idIOException
void commitPending(TaskAttemptID taskId, TaskStatus taskStatus) throws IOException, InterruptedException
taskId
- task's idtaskStatus
- status of the childIOException
InterruptedException
boolean canCommit(TaskAttemptID taskid) throws IOException
taskid
- IOException
void shuffleError(TaskAttemptID taskId, String message) throws IOException
IOException
void fsError(TaskAttemptID taskId, String message) throws IOException
IOException
void fatalError(TaskAttemptID taskId, String message, boolean fastFail) throws IOException
taskId
- task's idmessage
- fail messagefastFail
- flag to enable fast fail for taskIOException
MapTaskCompletionEventsUpdate getMapCompletionEvents(JobID jobId, int fromIndex, int maxLocs, TaskAttemptID id) throws IOException
fromIndex
- the index starting from which the locations should be
fetchedmaxLocs
- the max number of locations to fetchid
- The attempt id of the task that is trying to communicateMapTaskCompletionEventsUpdate
IOException
void preempted(TaskAttemptID taskId, TaskStatus taskStatus) throws IOException, InterruptedException
taskId
- task's idtaskStatus
- status of the childIOException
InterruptedException
TaskCheckpointID getCheckpointID(TaskID taskID)
taskID
- task's idvoid setCheckpointID(TaskID tid, TaskCheckpointID cid)
tid
- cid
- Copyright © 2008–2018 Apache Software Foundation. All rights reserved.