|
@@ -43,6 +43,7 @@ import org.apache.hadoop.fs.Path;
|
|
|
import org.apache.hadoop.io.Text;
|
|
|
import org.apache.hadoop.mapred.JobACLsManager;
|
|
|
import org.apache.hadoop.mapred.JobConf;
|
|
|
+import org.apache.hadoop.mapred.TaskCompletionEvent;
|
|
|
import org.apache.hadoop.mapreduce.Counters;
|
|
|
import org.apache.hadoop.mapreduce.JobACL;
|
|
|
import org.apache.hadoop.mapreduce.JobContext;
|
|
@@ -130,6 +131,9 @@ public class JobImpl implements org.apache.hadoop.mapreduce.v2.app.job.Job,
|
|
|
private static final TaskAttemptCompletionEvent[]
|
|
|
EMPTY_TASK_ATTEMPT_COMPLETION_EVENTS = new TaskAttemptCompletionEvent[0];
|
|
|
|
|
|
+ private static final TaskCompletionEvent[]
|
|
|
+ EMPTY_TASK_COMPLETION_EVENTS = new TaskCompletionEvent[0];
|
|
|
+
|
|
|
private static final Log LOG = LogFactory.getLog(JobImpl.class);
|
|
|
|
|
|
//The maximum fraction of fetch failures allowed for a map
|
|
@@ -196,7 +200,8 @@ public class JobImpl implements org.apache.hadoop.mapreduce.v2.app.job.Job,
|
|
|
private int allowedMapFailuresPercent = 0;
|
|
|
private int allowedReduceFailuresPercent = 0;
|
|
|
private List<TaskAttemptCompletionEvent> taskAttemptCompletionEvents;
|
|
|
- private List<TaskAttemptCompletionEvent> mapAttemptCompletionEvents;
|
|
|
+ private List<TaskCompletionEvent> mapAttemptCompletionEvents;
|
|
|
+ private List<Integer> taskCompletionIdxToMapCompletionIdx;
|
|
|
private final List<String> diagnostics = new ArrayList<String>();
|
|
|
|
|
|
//task/attempt related datastructures
|
|
@@ -684,27 +689,31 @@ public class JobImpl implements org.apache.hadoop.mapreduce.v2.app.job.Job,
|
|
|
@Override
|
|
|
public TaskAttemptCompletionEvent[] getTaskAttemptCompletionEvents(
|
|
|
int fromEventId, int maxEvents) {
|
|
|
- return getAttemptCompletionEvents(taskAttemptCompletionEvents,
|
|
|
- fromEventId, maxEvents);
|
|
|
+ TaskAttemptCompletionEvent[] events = EMPTY_TASK_ATTEMPT_COMPLETION_EVENTS;
|
|
|
+ readLock.lock();
|
|
|
+ try {
|
|
|
+ if (taskAttemptCompletionEvents.size() > fromEventId) {
|
|
|
+ int actualMax = Math.min(maxEvents,
|
|
|
+ (taskAttemptCompletionEvents.size() - fromEventId));
|
|
|
+ events = taskAttemptCompletionEvents.subList(fromEventId,
|
|
|
+ actualMax + fromEventId).toArray(events);
|
|
|
+ }
|
|
|
+ return events;
|
|
|
+ } finally {
|
|
|
+ readLock.unlock();
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
@Override
|
|
|
- public TaskAttemptCompletionEvent[] getMapAttemptCompletionEvents(
|
|
|
+ public TaskCompletionEvent[] getMapAttemptCompletionEvents(
|
|
|
int startIndex, int maxEvents) {
|
|
|
- return getAttemptCompletionEvents(mapAttemptCompletionEvents,
|
|
|
- startIndex, maxEvents);
|
|
|
- }
|
|
|
-
|
|
|
- private TaskAttemptCompletionEvent[] getAttemptCompletionEvents(
|
|
|
- List<TaskAttemptCompletionEvent> eventList,
|
|
|
- int startIndex, int maxEvents) {
|
|
|
- TaskAttemptCompletionEvent[] events = EMPTY_TASK_ATTEMPT_COMPLETION_EVENTS;
|
|
|
+ TaskCompletionEvent[] events = EMPTY_TASK_COMPLETION_EVENTS;
|
|
|
readLock.lock();
|
|
|
try {
|
|
|
- if (eventList.size() > startIndex) {
|
|
|
+ if (mapAttemptCompletionEvents.size() > startIndex) {
|
|
|
int actualMax = Math.min(maxEvents,
|
|
|
- (eventList.size() - startIndex));
|
|
|
- events = eventList.subList(startIndex,
|
|
|
+ (mapAttemptCompletionEvents.size() - startIndex));
|
|
|
+ events = mapAttemptCompletionEvents.subList(startIndex,
|
|
|
actualMax + startIndex).toArray(events);
|
|
|
}
|
|
|
return events;
|
|
@@ -1247,7 +1256,9 @@ public class JobImpl implements org.apache.hadoop.mapreduce.v2.app.job.Job,
|
|
|
new ArrayList<TaskAttemptCompletionEvent>(
|
|
|
job.numMapTasks + job.numReduceTasks + 10);
|
|
|
job.mapAttemptCompletionEvents =
|
|
|
- new ArrayList<TaskAttemptCompletionEvent>(job.numMapTasks + 10);
|
|
|
+ new ArrayList<TaskCompletionEvent>(job.numMapTasks + 10);
|
|
|
+ job.taskCompletionIdxToMapCompletionIdx = new ArrayList<Integer>(
|
|
|
+ job.numMapTasks + job.numReduceTasks + 10);
|
|
|
|
|
|
job.allowedMapFailuresPercent =
|
|
|
job.conf.getInt(MRJobConfig.MAP_FAILURES_MAX_PERCENT, 0);
|
|
@@ -1562,19 +1573,37 @@ public class JobImpl implements org.apache.hadoop.mapreduce.v2.app.job.Job,
|
|
|
//eventId is equal to index in the arraylist
|
|
|
tce.setEventId(job.taskAttemptCompletionEvents.size());
|
|
|
job.taskAttemptCompletionEvents.add(tce);
|
|
|
+ int mapEventIdx = -1;
|
|
|
if (TaskType.MAP.equals(tce.getAttemptId().getTaskId().getTaskType())) {
|
|
|
- job.mapAttemptCompletionEvents.add(tce);
|
|
|
+ // we track map completions separately from task completions because
|
|
|
+ // - getMapAttemptCompletionEvents uses index ranges specific to maps
|
|
|
+ // - type converting the same events over and over is expensive
|
|
|
+ mapEventIdx = job.mapAttemptCompletionEvents.size();
|
|
|
+ job.mapAttemptCompletionEvents.add(TypeConverter.fromYarn(tce));
|
|
|
}
|
|
|
+ job.taskCompletionIdxToMapCompletionIdx.add(mapEventIdx);
|
|
|
|
|
|
TaskAttemptId attemptId = tce.getAttemptId();
|
|
|
TaskId taskId = attemptId.getTaskId();
|
|
|
//make the previous completion event as obsolete if it exists
|
|
|
- Object successEventNo =
|
|
|
- job.successAttemptCompletionEventNoMap.remove(taskId);
|
|
|
+ Integer successEventNo =
|
|
|
+ job.successAttemptCompletionEventNoMap.remove(taskId);
|
|
|
if (successEventNo != null) {
|
|
|
TaskAttemptCompletionEvent successEvent =
|
|
|
- job.taskAttemptCompletionEvents.get((Integer) successEventNo);
|
|
|
+ job.taskAttemptCompletionEvents.get(successEventNo);
|
|
|
successEvent.setStatus(TaskAttemptCompletionEventStatus.OBSOLETE);
|
|
|
+ int mapCompletionIdx =
|
|
|
+ job.taskCompletionIdxToMapCompletionIdx.get(successEventNo);
|
|
|
+ if (mapCompletionIdx >= 0) {
|
|
|
+ // update the corresponding TaskCompletionEvent for the map
|
|
|
+ TaskCompletionEvent mapEvent =
|
|
|
+ job.mapAttemptCompletionEvents.get(mapCompletionIdx);
|
|
|
+ job.mapAttemptCompletionEvents.set(mapCompletionIdx,
|
|
|
+ new TaskCompletionEvent(mapEvent.getEventId(),
|
|
|
+ mapEvent.getTaskAttemptId(), mapEvent.idWithinJob(),
|
|
|
+ mapEvent.isMapTask(), TaskCompletionEvent.Status.OBSOLETE,
|
|
|
+ mapEvent.getTaskTrackerHttp()));
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
// if this attempt is not successful then why is the previous successful
|