|
@@ -49,8 +49,6 @@ import org.apache.hadoop.yarn.api.records.UpdateContainerRequest;
|
|
|
import org.apache.hadoop.yarn.client.api.AMRMClient;
|
|
|
import org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
|
-import org.apache.hadoop.yarn.event.Dispatcher;
|
|
|
-import org.apache.hadoop.yarn.event.DrainDispatcher;
|
|
|
import org.apache.hadoop.yarn.event.EventHandler;
|
|
|
import org.apache.hadoop.yarn.ipc.YarnRPC;
|
|
|
import org.apache.hadoop.yarn.security.AMRMTokenIdentifier;
|
|
@@ -126,22 +124,20 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// Phase-1 Start 1st RM
|
|
|
MyResourceManager rm1 = new MyResourceManager(conf, memStore);
|
|
|
rm1.start();
|
|
|
- DrainDispatcher dispatcher =
|
|
|
- (DrainDispatcher) rm1.getRMContext().getDispatcher();
|
|
|
|
|
|
// Submit the application
|
|
|
RMApp app = rm1.submitApp(1024);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
MockNM nm1 = new MockNM("h1:1234", 15120, rm1.getResourceTrackerService());
|
|
|
nm1.registerNode();
|
|
|
nm1.nodeHeartbeat(true); // Node heartbeat
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
ApplicationAttemptId appAttemptId =
|
|
|
app.getCurrentAppAttempt().getAppAttemptId();
|
|
|
rm1.sendAMLaunched(appAttemptId);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
org.apache.hadoop.security.token.Token<AMRMTokenIdentifier> token =
|
|
|
rm1.getRMContext().getRMApps().get(appAttemptId.getApplicationId())
|
|
@@ -176,7 +172,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
blacklistAdditions.remove("h2");// remove from local list
|
|
|
|
|
|
AllocateResponse allocateResponse = amClient.allocate(0.1f);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
Assert.assertEquals("No of assignments must be 0", 0, allocateResponse
|
|
|
.getAllocatedContainers().size());
|
|
|
|
|
@@ -189,10 +185,10 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// Step-2 : NM heart beat is sent.
|
|
|
// On 2nd AM allocate request, RM allocates 3 containers to AM
|
|
|
nm1.nodeHeartbeat(true); // Node heartbeat
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
allocateResponse = amClient.allocate(0.2f);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
// 3 containers are allocated i.e for cRequest1, cRequest2 and cRequest3.
|
|
|
Assert.assertEquals("No of assignments must be 0", 3, allocateResponse
|
|
|
.getAllocatedContainers().size());
|
|
@@ -207,7 +203,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
amClient.removeContainerRequest(cRequest3);
|
|
|
|
|
|
allocateResponse = amClient.allocate(0.2f);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
Assert.assertEquals("No of assignments must be 0", 0, allocateResponse
|
|
|
.getAllocatedContainers().size());
|
|
|
assertAsksAndReleases(4, 0, rm1);
|
|
@@ -233,7 +229,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// request
|
|
|
nm1.nodeHeartbeat(containerId.getApplicationAttemptId(),
|
|
|
containerId.getContainerId(), ContainerState.RUNNING);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
amClient.requestContainerUpdate(
|
|
|
container, UpdateContainerRequest.newInstance(
|
|
|
container.getVersion(), container.getId(),
|
|
@@ -242,7 +238,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
it.remove();
|
|
|
|
|
|
allocateResponse = amClient.allocate(0.3f);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
Assert.assertEquals("No of assignments must be 0", 0, allocateResponse
|
|
|
.getAllocatedContainers().size());
|
|
|
assertAsksAndReleases(3, pendingRelease, rm1);
|
|
@@ -258,7 +254,6 @@ public class TestAMRMClientOnRMRestart {
|
|
|
rm2.start();
|
|
|
nm1.setResourceTrackerService(rm2.getResourceTrackerService());
|
|
|
((MyAMRMClientImpl) amClient).updateRMProxy(rm2);
|
|
|
- dispatcher = (DrainDispatcher) rm2.getRMContext().getDispatcher();
|
|
|
|
|
|
// NM should be rebooted on heartbeat, even first heartbeat for nm2
|
|
|
NodeHeartbeatResponse hbResponse = nm1.nodeHeartbeat(true);
|
|
@@ -274,7 +269,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
Collections.singletonList(
|
|
|
containerId.getApplicationAttemptId().getApplicationId()));
|
|
|
nm1.nodeHeartbeat(true);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
|
|
|
blacklistAdditions.add("h3");
|
|
|
amClient.updateBlacklist(blacklistAdditions, null);
|
|
@@ -296,7 +291,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// containerRequest and blacklisted nodes.
|
|
|
// Intern RM send resync command,AMRMClient resend allocate request
|
|
|
allocateResponse = amClient.allocate(0.3f);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
|
|
|
completedContainer =
|
|
|
allocateResponse.getCompletedContainersStatuses().size();
|
|
@@ -313,7 +308,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
|
|
|
// Step-5 : Allocater after resync command
|
|
|
allocateResponse = amClient.allocate(0.5f);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
Assert.assertEquals("No of assignments must be 0", 0, allocateResponse
|
|
|
.getAllocatedContainers().size());
|
|
|
|
|
@@ -326,10 +321,10 @@ public class TestAMRMClientOnRMRestart {
|
|
|
int count = 5;
|
|
|
while (count-- > 0) {
|
|
|
nm1.nodeHeartbeat(true);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
|
|
|
allocateResponse = amClient.allocate(0.5f);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
noAssignedContainer += allocateResponse.getAllocatedContainers().size();
|
|
|
if (noAssignedContainer == 3) {
|
|
|
break;
|
|
@@ -358,22 +353,20 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// Phase-1 Start 1st RM
|
|
|
MyResourceManager rm1 = new MyResourceManager(conf, memStore);
|
|
|
rm1.start();
|
|
|
- DrainDispatcher dispatcher =
|
|
|
- (DrainDispatcher) rm1.getRMContext().getDispatcher();
|
|
|
|
|
|
// Submit the application
|
|
|
RMApp app = rm1.submitApp(1024);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
MockNM nm1 = new MockNM("h1:1234", 15120, rm1.getResourceTrackerService());
|
|
|
nm1.registerNode();
|
|
|
nm1.nodeHeartbeat(true); // Node heartbeat
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
ApplicationAttemptId appAttemptId =
|
|
|
app.getCurrentAppAttempt().getAppAttemptId();
|
|
|
rm1.sendAMLaunched(appAttemptId);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
org.apache.hadoop.security.token.Token<AMRMTokenIdentifier> token =
|
|
|
rm1.getRMContext().getRMApps().get(appAttemptId.getApplicationId())
|
|
@@ -393,7 +386,6 @@ public class TestAMRMClientOnRMRestart {
|
|
|
rm2.start();
|
|
|
nm1.setResourceTrackerService(rm2.getResourceTrackerService());
|
|
|
((MyAMRMClientImpl) amClient).updateRMProxy(rm2);
|
|
|
- dispatcher = (DrainDispatcher) rm2.getRMContext().getDispatcher();
|
|
|
|
|
|
// NM should be rebooted on heartbeat, even first heartbeat for nm2
|
|
|
NodeHeartbeatResponse hbResponse = nm1.nodeHeartbeat(true);
|
|
@@ -409,7 +401,7 @@ public class TestAMRMClientOnRMRestart {
|
|
|
Priority.newInstance(0), 0);
|
|
|
nm1.registerNode(Arrays.asList(containerReport), null);
|
|
|
nm1.nodeHeartbeat(true);
|
|
|
- dispatcher.await();
|
|
|
+ rm2.drainEvents();
|
|
|
|
|
|
amClient.unregisterApplicationMaster(FinalApplicationStatus.SUCCEEDED,
|
|
|
null, null);
|
|
@@ -421,7 +413,6 @@ public class TestAMRMClientOnRMRestart {
|
|
|
amClient.stop();
|
|
|
rm1.stop();
|
|
|
rm2.stop();
|
|
|
-
|
|
|
}
|
|
|
|
|
|
|
|
@@ -439,22 +430,20 @@ public class TestAMRMClientOnRMRestart {
|
|
|
// start first RM
|
|
|
MyResourceManager2 rm1 = new MyResourceManager2(conf, memStore);
|
|
|
rm1.start();
|
|
|
- DrainDispatcher dispatcher =
|
|
|
- (DrainDispatcher) rm1.getRMContext().getDispatcher();
|
|
|
Long startTime = System.currentTimeMillis();
|
|
|
// Submit the application
|
|
|
RMApp app = rm1.submitApp(1024);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
MockNM nm1 = new MockNM("h1:1234", 15120, rm1.getResourceTrackerService());
|
|
|
nm1.registerNode();
|
|
|
nm1.nodeHeartbeat(true); // Node heartbeat
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
ApplicationAttemptId appAttemptId =
|
|
|
app.getCurrentAppAttempt().getAppAttemptId();
|
|
|
rm1.sendAMLaunched(appAttemptId);
|
|
|
- dispatcher.await();
|
|
|
+ rm1.drainEvents();
|
|
|
|
|
|
AMRMTokenSecretManager amrmTokenSecretManagerForRM1 =
|
|
|
rm1.getRMContext().getAMRMTokenSecretManager();
|
|
@@ -513,7 +502,6 @@ public class TestAMRMClientOnRMRestart {
|
|
|
rm2.start();
|
|
|
nm1.setResourceTrackerService(rm2.getResourceTrackerService());
|
|
|
((MyAMRMClientImpl) amClient).updateRMProxy(rm2);
|
|
|
- dispatcher = (DrainDispatcher) rm2.getRMContext().getDispatcher();
|
|
|
|
|
|
AMRMTokenSecretManager amrmTokenSecretManagerForRM2 =
|
|
|
rm2.getRMContext().getAMRMTokenSecretManager();
|
|
@@ -615,11 +603,6 @@ public class TestAMRMClientOnRMRestart {
|
|
|
MyResourceManager.setClusterTimeStamp(fakeClusterTimeStamp);
|
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
- protected Dispatcher createDispatcher() {
|
|
|
- return new DrainDispatcher();
|
|
|
- }
|
|
|
-
|
|
|
@Override
|
|
|
protected EventHandler<SchedulerEvent> createSchedulerEventDispatcher() {
|
|
|
// Dispatch inline for test sanity
|