|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |
public interface TaskUmbilicalProtocol
Protocol that task child process uses to contact its parent process. The parent is a daemon which which polls the central master for a new map or reduce task and runs it as a child process. All communication between child and parent is via this protocol.
Field Summary | |
---|---|
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). |
Method Summary | |
---|---|
boolean |
canCommit(TaskAttemptID taskid,
org.apache.hadoop.mapred.JvmContext jvmContext)
Polling to know whether the task can go-ahead with commit |
void |
commitPending(TaskAttemptID taskId,
TaskStatus taskStatus,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report that the task is complete, but its commit is pending. |
void |
done(TaskAttemptID taskid,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report that the task is successfully completed. |
void |
fatalError(TaskAttemptID taskId,
String message,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report that the task encounted a fatal error. |
void |
fsError(TaskAttemptID taskId,
String message,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report that the task encounted a local filesystem error. |
MapTaskCompletionEventsUpdate |
getMapCompletionEvents(JobID jobId,
int fromIndex,
int maxLocs,
TaskAttemptID id,
org.apache.hadoop.mapred.JvmContext jvmContext)
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. |
boolean |
ping(TaskAttemptID taskid,
org.apache.hadoop.mapred.JvmContext jvmContext)
Periodically called by child to check if parent is still alive. |
void |
reportDiagnosticInfo(TaskAttemptID taskid,
String trace,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report error messages back to parent. |
void |
reportNextRecordRange(TaskAttemptID taskid,
org.apache.hadoop.mapred.SortedRanges.Range range,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report the record range which is going to process next by the Task. |
void |
shuffleError(TaskAttemptID taskId,
String message,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report that a reduce-task couldn't shuffle map-outputs. |
boolean |
statusUpdate(TaskAttemptID taskId,
TaskStatus taskStatus,
org.apache.hadoop.mapred.JvmContext jvmContext)
Report child's progress to parent. |
void |
updatePrivateDistributedCacheSizes(JobID jobId,
long[] sizes)
The job initializer needs to report the sizes of the archive objects and directories in the private distributed cache. |
Methods inherited from interface org.apache.hadoop.ipc.VersionedProtocol |
---|
getProtocolVersion |
Field Detail |
---|
static final long versionID
Method Detail |
---|
JvmTask getTask(org.apache.hadoop.mapred.JvmContext context) throws IOException
context
- the JvmContext of the JVM w.r.t the TaskTracker that
launched it
IOException
boolean statusUpdate(TaskAttemptID taskId, TaskStatus taskStatus, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException, InterruptedException
taskId
- task-id of the childtaskStatus
- status of the childjvmContext
- context the jvmContext running the task.
IOException
InterruptedException
void reportDiagnosticInfo(TaskAttemptID taskid, String trace, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
taskid
- the id of the task involvedtrace
- the text to reportjvmContext
- context the jvmContext running the task.
IOException
void reportNextRecordRange(TaskAttemptID taskid, org.apache.hadoop.mapred.SortedRanges.Range range, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
taskid
- the id of the task involvedrange
- the range of record sequence nosjvmContext
- context the jvmContext running the task.
IOException
boolean ping(TaskAttemptID taskid, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
taskid
- the id of the task involvedjvmContext
- context the jvmContext running the task.
IOException
void done(TaskAttemptID taskid, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
taskid
- task's idjvmContext
- context the jvmContext running the task.
IOException
void commitPending(TaskAttemptID taskId, TaskStatus taskStatus, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException, InterruptedException
taskId
- task's idtaskStatus
- status of the childjvmContext
- context the jvmContext running the task.
IOException
InterruptedException
boolean canCommit(TaskAttemptID taskid, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
taskid
- jvmContext
- context the jvmContext running the task.
IOException
void shuffleError(TaskAttemptID taskId, String message, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
IOException
void fsError(TaskAttemptID taskId, String message, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
IOException
void fatalError(TaskAttemptID taskId, String message, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
IOException
MapTaskCompletionEventsUpdate getMapCompletionEvents(JobID jobId, int fromIndex, int maxLocs, TaskAttemptID id, org.apache.hadoop.mapred.JvmContext jvmContext) throws IOException
jobId
- the reducer job idfromIndex
- 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 communicate
MapTaskCompletionEventsUpdate
IOException
void updatePrivateDistributedCacheSizes(JobID jobId, long[] sizes) throws IOException
jobId
- the job to updatesizes
- the array of sizes that were computed
IOException
|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |