|
@@ -24,6 +24,7 @@ import static org.mockito.Matchers.anyInt;
|
|
|
import static org.mockito.Matchers.isA;
|
|
|
import static org.mockito.Mockito.doCallRealMethod;
|
|
|
import static org.mockito.Mockito.doReturn;
|
|
|
+import static org.mockito.Mockito.inOrder;
|
|
|
import static org.mockito.Mockito.mock;
|
|
|
import static org.mockito.Mockito.never;
|
|
|
import static org.mockito.Mockito.times;
|
|
@@ -70,11 +71,13 @@ import org.apache.hadoop.mapreduce.v2.app.job.event.JobEvent;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.JobEventType;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.JobUpdatedNodesEvent;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.TaskAttemptContainerAssignedEvent;
|
|
|
+import org.apache.hadoop.mapreduce.v2.app.job.event.TaskAttemptDiagnosticsUpdateEvent;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.TaskAttemptEvent;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.TaskAttemptEventType;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.event.TaskAttemptKillEvent;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.impl.JobImpl;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.job.impl.TaskAttemptImpl;
|
|
|
+import org.apache.hadoop.mapreduce.v2.app.rm.preemption.AMPreemptionPolicy;
|
|
|
import org.apache.hadoop.mapreduce.v2.app.rm.preemption.NoopAMPreemptionPolicy;
|
|
|
import org.apache.hadoop.mapreduce.v2.util.MRBuilderUtils;
|
|
|
import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
|
|
@@ -144,6 +147,7 @@ import org.junit.Before;
|
|
|
import org.junit.Test;
|
|
|
|
|
|
import com.google.common.base.Supplier;
|
|
|
+import org.mockito.InOrder;
|
|
|
|
|
|
@SuppressWarnings("unchecked")
|
|
|
public class TestRMContainerAllocator {
|
|
@@ -3017,6 +3021,48 @@ public class TestRMContainerAllocator {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
+ /**
|
|
|
+ * MAPREDUCE-6771. Test if RMContainerAllocator generates the events in the
|
|
|
+ * right order while processing finished containers.
|
|
|
+ */
|
|
|
+ @Test
|
|
|
+ public void testHandlingFinishedContainers() {
|
|
|
+ EventHandler eventHandler = mock(EventHandler.class);
|
|
|
+
|
|
|
+ AppContext context = mock(RunningAppContext.class);
|
|
|
+ when(context.getClock()).thenReturn(new ControlledClock());
|
|
|
+ when(context.getClusterInfo()).thenReturn(
|
|
|
+ new ClusterInfo(Resource.newInstance(10240, 1)));
|
|
|
+ when(context.getEventHandler()).thenReturn(eventHandler);
|
|
|
+ RMContainerAllocator containerAllocator =
|
|
|
+ new RMContainerAllocatorForFinishedContainer(null, context,
|
|
|
+ mock(AMPreemptionPolicy.class));
|
|
|
+
|
|
|
+ ContainerStatus finishedContainer = ContainerStatus.newInstance(
|
|
|
+ mock(ContainerId.class), ContainerState.COMPLETE, "", 0);
|
|
|
+ containerAllocator.processFinishedContainer(finishedContainer);
|
|
|
+
|
|
|
+ InOrder inOrder = inOrder(eventHandler);
|
|
|
+ inOrder.verify(eventHandler).handle(
|
|
|
+ isA(TaskAttemptDiagnosticsUpdateEvent.class));
|
|
|
+ inOrder.verify(eventHandler).handle(isA(TaskAttemptEvent.class));
|
|
|
+ inOrder.verifyNoMoreInteractions();
|
|
|
+ }
|
|
|
+
|
|
|
+ private static class RMContainerAllocatorForFinishedContainer
|
|
|
+ extends RMContainerAllocator {
|
|
|
+ public RMContainerAllocatorForFinishedContainer(ClientService clientService,
|
|
|
+ AppContext context, AMPreemptionPolicy preemptionPolicy) {
|
|
|
+ super(clientService, context, preemptionPolicy);
|
|
|
+ }
|
|
|
+ @Override
|
|
|
+ protected AssignedRequests createAssignedRequests() {
|
|
|
+ AssignedRequests assignedReqs = mock(AssignedRequests.class);
|
|
|
+ TaskAttemptId taskAttempt = mock(TaskAttemptId.class);
|
|
|
+ when(assignedReqs.get(any(ContainerId.class))).thenReturn(taskAttempt);
|
|
|
+ return assignedReqs;
|
|
|
+ }
|
|
|
+ }
|
|
|
|
|
|
@Test
|
|
|
public void testAvoidAskMoreReducersWhenReducerPreemptionIsRequired()
|