|
@@ -18,13 +18,18 @@
|
|
|
|
|
|
package org.apache.hadoop.yarn.server.resourcemanager.rmapp;
|
|
|
|
|
|
-import static org.mockito.Mockito.mock;
|
|
|
+import static org.junit.Assert.assertEquals;
|
|
|
import static org.junit.Assume.assumeTrue;
|
|
|
+import static org.mockito.Matchers.any;
|
|
|
+import static org.mockito.Mockito.mock;
|
|
|
import static org.mockito.Mockito.verify;
|
|
|
+import static org.mockito.Mockito.when;
|
|
|
|
|
|
import java.io.IOException;
|
|
|
import java.util.Arrays;
|
|
|
import java.util.Collection;
|
|
|
+import java.util.Collections;
|
|
|
+import java.util.List;
|
|
|
|
|
|
import junit.framework.Assert;
|
|
|
|
|
@@ -35,10 +40,13 @@ import org.apache.hadoop.security.SecurityUtil;
|
|
|
import org.apache.hadoop.security.UserGroupInformation;
|
|
|
import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod;
|
|
|
import org.apache.hadoop.yarn.MockApps;
|
|
|
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
|
|
|
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
|
|
import org.apache.hadoop.yarn.api.records.ApplicationReport;
|
|
|
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
|
|
+import org.apache.hadoop.yarn.api.records.Container;
|
|
|
import org.apache.hadoop.yarn.api.records.FinalApplicationStatus;
|
|
|
+import org.apache.hadoop.yarn.api.records.Resource;
|
|
|
import org.apache.hadoop.yarn.api.records.impl.pb.ApplicationSubmissionContextPBImpl;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
|
import org.apache.hadoop.yarn.event.DrainDispatcher;
|
|
@@ -54,7 +62,10 @@ import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.event.RMAppAttemptContainerAllocatedEvent;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.event.RMAppAttemptStoredEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.Allocation;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.YarnScheduler;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
|
@@ -62,7 +73,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.security.AMRMTokenSecretMan
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.security.ClientToAMTokenSecretManagerInRM;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.security.NMTokenSecretManagerInRM;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSecretManager;
|
|
|
-import org.junit.After;
|
|
|
+import org.apache.hadoop.yarn.server.utils.BuilderUtils;
|
|
|
import org.junit.Before;
|
|
|
import org.junit.Test;
|
|
|
import org.junit.runner.RunWith;
|
|
@@ -81,6 +92,7 @@ public class TestRMAppTransitions {
|
|
|
private static int appId = 1;
|
|
|
private DrainDispatcher rmDispatcher;
|
|
|
private RMStateStore store;
|
|
|
+ private YarnScheduler scheduler;
|
|
|
|
|
|
// ignore all the RM application attempt events
|
|
|
private static final class TestApplicationAttemptEventDispatcher implements
|
|
@@ -206,7 +218,8 @@ public class TestRMAppTransitions {
|
|
|
String queue = MockApps.newQueue();
|
|
|
// ensure max application attempts set to known value
|
|
|
conf.setInt(YarnConfiguration.RM_AM_MAX_ATTEMPTS, maxAppAttempts);
|
|
|
- YarnScheduler scheduler = mock(YarnScheduler.class);
|
|
|
+ scheduler = mock(YarnScheduler.class);
|
|
|
+
|
|
|
ApplicationMasterService masterService =
|
|
|
new ApplicationMasterService(rmContext, scheduler);
|
|
|
|
|
@@ -392,8 +405,7 @@ public class TestRMAppTransitions {
|
|
|
RMApp application = testCreateAppRemoving(submissionContext);
|
|
|
// REMOVING => FINISHING event RMAppEventType.APP_REMOVED
|
|
|
RMAppEvent finishingEvent =
|
|
|
- new RMAppEvent(application.getApplicationId(),
|
|
|
- RMAppEventType.APP_REMOVED);
|
|
|
+ new RMAppRemovedEvent(application.getApplicationId(), null);
|
|
|
application.handle(finishingEvent);
|
|
|
assertAppState(RMAppState.FINISHING, application);
|
|
|
assertTimesAtFinish(application);
|
|
@@ -816,6 +828,14 @@ public class TestRMAppTransitions {
|
|
|
app = testCreateAppRunning(null);
|
|
|
rmDispatcher.await();
|
|
|
assertAppState(RMAppState.RUNNING, app);
|
|
|
+
|
|
|
+ report = app.createAndGetApplicationReport("clientuser", true);
|
|
|
+ Assert.assertNull(report.getClientToAMToken());
|
|
|
+
|
|
|
+ // this method is to make AMLaunchedTransition invoked inside which
|
|
|
+ // ClientTokenMasterKey is registered in ClientTokenSecretManager
|
|
|
+ moveCurrentAttemptToLaunchedState(app.getCurrentAppAttempt());
|
|
|
+
|
|
|
report = app.createAndGetApplicationReport(null, true);
|
|
|
Assert.assertNull(report.getClientToAMToken());
|
|
|
report = app.createAndGetApplicationReport("clientuser", true);
|
|
@@ -830,4 +850,33 @@ public class TestRMAppTransitions {
|
|
|
report = app.createAndGetApplicationReport("clientuser", true);
|
|
|
Assert.assertNull(report.getClientToAMToken());
|
|
|
}
|
|
|
+
|
|
|
+ @SuppressWarnings("unchecked")
|
|
|
+ private void moveCurrentAttemptToLaunchedState(RMAppAttempt attempt) {
|
|
|
+ attempt.handle(new RMAppAttemptEvent(attempt.getAppAttemptId(),
|
|
|
+ RMAppAttemptEventType.APP_ACCEPTED));
|
|
|
+ // Mock the allocation of AM container
|
|
|
+ Container container = mock(Container.class);
|
|
|
+ Resource resource = BuilderUtils.newResource(2048, 1);
|
|
|
+ when(container.getId()).thenReturn(
|
|
|
+ BuilderUtils.newContainerId(attempt.getAppAttemptId(), 1));
|
|
|
+ when(container.getResource()).thenReturn(resource);
|
|
|
+ Allocation allocation = mock(Allocation.class);
|
|
|
+ when(allocation.getContainers()).thenReturn(
|
|
|
+ Collections.singletonList(container));
|
|
|
+ when(allocation.getContainers()).
|
|
|
+ thenReturn(Collections.singletonList(container));
|
|
|
+ when(
|
|
|
+ scheduler.allocate(any(ApplicationAttemptId.class), any(List.class),
|
|
|
+ any(List.class), any(List.class), any(List.class))).thenReturn(
|
|
|
+ allocation);
|
|
|
+ attempt.handle(new RMAppAttemptContainerAllocatedEvent(attempt
|
|
|
+ .getAppAttemptId(), container));
|
|
|
+ attempt
|
|
|
+ .handle(new RMAppAttemptStoredEvent(attempt.getAppAttemptId(), null));
|
|
|
+ attempt.handle(new RMAppAttemptEvent(attempt.getAppAttemptId(),
|
|
|
+ RMAppAttemptEventType.LAUNCHED));
|
|
|
+
|
|
|
+ assertEquals(RMAppAttemptState.LAUNCHED, attempt.getAppAttemptState());
|
|
|
+ }
|
|
|
}
|