|
@@ -47,6 +47,7 @@ import static org.junit.Assert.assertEquals;
|
|
|
import static org.junit.Assert.assertFalse;
|
|
|
import static org.junit.Assert.assertTrue;
|
|
|
import static org.junit.Assert.fail;
|
|
|
+import static org.junit.Assert.assertThrows;
|
|
|
import static org.mockito.Mockito.any;
|
|
|
import static org.mockito.Mockito.anyLong;
|
|
|
import static org.mockito.Mockito.doAnswer;
|
|
@@ -473,7 +474,8 @@ public class TestBalancer {
|
|
|
static void waitForBalancer(long totalUsedSpace, long totalCapacity,
|
|
|
ClientProtocol client, MiniDFSCluster cluster, BalancerParameters p,
|
|
|
int expectedExcludedNodes) throws IOException, TimeoutException {
|
|
|
- waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p, expectedExcludedNodes, true);
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client,
|
|
|
+ cluster, p, expectedExcludedNodes, true);
|
|
|
}
|
|
|
|
|
|
/**
|
|
@@ -495,6 +497,9 @@ public class TestBalancer {
|
|
|
if (!p.getExcludedNodes().isEmpty()) {
|
|
|
totalCapacity -= p.getExcludedNodes().size() * CAPACITY;
|
|
|
}
|
|
|
+ if (!p.getExcludedTargetNodes().isEmpty()) {
|
|
|
+ totalCapacity -= p.getExcludedTargetNodes().size() * CAPACITY;
|
|
|
+ }
|
|
|
final double avgUtilization = ((double)totalUsedSpace) / totalCapacity;
|
|
|
boolean balanced;
|
|
|
do {
|
|
@@ -539,6 +544,69 @@ public class TestBalancer {
|
|
|
} while (!balanced);
|
|
|
}
|
|
|
|
|
|
+ /**
|
|
|
+ * Wait until balanced: each datanode gives utilization within.
|
|
|
+ * Used when testing for included / excluded target and source nodes.
|
|
|
+ * BALANCE_ALLOWED_VARIANCE of average
|
|
|
+ * @throws IOException
|
|
|
+ * @throws TimeoutException
|
|
|
+ */
|
|
|
+ static void waitForBalancer(long totalUsedSpace, long totalCapacity,
|
|
|
+ ClientProtocol client, MiniDFSCluster cluster, BalancerParameters p,
|
|
|
+ int expectedExcludedSourceNodes, int expectedExcludedTargetNodes)
|
|
|
+ throws IOException, TimeoutException {
|
|
|
+ long timeout = TIMEOUT;
|
|
|
+ long failtime = (timeout <= 0L) ? Long.MAX_VALUE
|
|
|
+ : Time.monotonicNow() + timeout;
|
|
|
+ if (!p.getExcludedTargetNodes().isEmpty()) {
|
|
|
+ totalCapacity -= p.getExcludedTargetNodes().size() * CAPACITY;
|
|
|
+ }
|
|
|
+ final double avgUtilization = ((double)totalUsedSpace) / totalCapacity;
|
|
|
+ boolean balanced;
|
|
|
+ do {
|
|
|
+ DatanodeInfo[] datanodeReport =
|
|
|
+ client.getDatanodeReport(DatanodeReportType.ALL);
|
|
|
+ assertEquals(datanodeReport.length, cluster.getDataNodes().size());
|
|
|
+ balanced = true;
|
|
|
+ int actualExcludedSourceNodeCount = 0;
|
|
|
+ int actualExcludedTargetNodeCount = 0;
|
|
|
+ for (DatanodeInfo datanode : datanodeReport) {
|
|
|
+ double nodeUtilization =
|
|
|
+ ((double) datanode.getDfsUsed() + datanode.getNonDfsUsed()) /
|
|
|
+ datanode.getCapacity();
|
|
|
+ if(Dispatcher.Util.isExcluded(p.getExcludedTargetNodes(), datanode)) {
|
|
|
+ actualExcludedTargetNodeCount++;
|
|
|
+ }
|
|
|
+ if(!Dispatcher.Util.isIncluded(p.getTargetNodes(), datanode)) {
|
|
|
+ actualExcludedTargetNodeCount++;
|
|
|
+ }
|
|
|
+ if(Dispatcher.Util.isExcluded(p.getExcludedSourceNodes(), datanode)) {
|
|
|
+ actualExcludedSourceNodeCount++;
|
|
|
+ }
|
|
|
+ if(!Dispatcher.Util.isIncluded(p.getSourceNodes(), datanode)) {
|
|
|
+ actualExcludedSourceNodeCount++;
|
|
|
+ }
|
|
|
+ if (Math.abs(avgUtilization - nodeUtilization) > BALANCE_ALLOWED_VARIANCE) {
|
|
|
+ balanced = false;
|
|
|
+ if (Time.monotonicNow() > failtime) {
|
|
|
+ throw new TimeoutException(
|
|
|
+ "Rebalancing expected avg utilization to become "
|
|
|
+ + avgUtilization + ", but on datanode " + datanode
|
|
|
+ + " it remains at " + nodeUtilization
|
|
|
+ + " after more than " + TIMEOUT + " msec.");
|
|
|
+ }
|
|
|
+ try {
|
|
|
+ Thread.sleep(100);
|
|
|
+ } catch (InterruptedException ignored) {
|
|
|
+ }
|
|
|
+ break;
|
|
|
+ }
|
|
|
+ }
|
|
|
+ assertEquals(expectedExcludedSourceNodes, actualExcludedSourceNodeCount);
|
|
|
+ assertEquals(expectedExcludedTargetNodes, actualExcludedTargetNodeCount);
|
|
|
+ } while (!balanced);
|
|
|
+ }
|
|
|
+
|
|
|
String long2String(long[] array) {
|
|
|
if (array.length == 0) {
|
|
|
return "<empty>";
|
|
@@ -636,6 +704,14 @@ public class TestBalancer {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
+ private void doTest(Configuration conf,
|
|
|
+ long newCapacity, String newRack, NewNodeInfo nodes,
|
|
|
+ boolean useTool, boolean useFile, BalancerParameters p) throws Exception {
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
+ newCapacity, 0L, newRack, nodes,
|
|
|
+ useTool, useFile, false, 0.3, p);
|
|
|
+ }
|
|
|
+
|
|
|
private void doTest(Configuration conf, long[] capacities, String[] racks,
|
|
|
long newCapacity, String newRack, boolean useTool) throws Exception {
|
|
|
doTest(conf, capacities, racks, newCapacity, newRack, null, useTool, false);
|
|
@@ -645,7 +721,7 @@ public class TestBalancer {
|
|
|
long newCapacity, String newRack, NewNodeInfo nodes,
|
|
|
boolean useTool, boolean useFile) throws Exception {
|
|
|
doTest(conf, capacities, racks, newCapacity, 0L, newRack, nodes,
|
|
|
- useTool, useFile, false, 0.3);
|
|
|
+ useTool, useFile, false, 0.3, null);
|
|
|
}
|
|
|
|
|
|
/** This test start a cluster with specified number of nodes,
|
|
@@ -671,7 +747,7 @@ public class TestBalancer {
|
|
|
private void doTest(Configuration conf, long[] capacities,
|
|
|
String[] racks, long newCapacity, long newNonDfsUsed, String newRack,
|
|
|
NewNodeInfo nodes, boolean useTool, boolean useFile,
|
|
|
- boolean useNamesystemSpy, double clusterUtilization) throws Exception {
|
|
|
+ boolean useNamesystemSpy, double clusterUtilization, BalancerParameters p) throws Exception {
|
|
|
LOG.info("capacities = " + long2String(capacities));
|
|
|
LOG.info("racks = " + Arrays.asList(racks));
|
|
|
LOG.info("newCapacity= " + newCapacity);
|
|
@@ -746,7 +822,7 @@ public class TestBalancer {
|
|
|
totalNodes-1-i).getDatanodeId().getXferAddr());
|
|
|
}
|
|
|
}
|
|
|
- //polulate the exclude nodes
|
|
|
+ //populate the exclude nodes
|
|
|
if (nodes.getNumberofExcludeNodes() > 0) {
|
|
|
int totalNodes = cluster.getDataNodes().size();
|
|
|
for (int i=0; i < nodes.getNumberofExcludeNodes(); i++) {
|
|
@@ -756,16 +832,16 @@ public class TestBalancer {
|
|
|
}
|
|
|
}
|
|
|
}
|
|
|
- // run balancer and validate results
|
|
|
- BalancerParameters.Builder pBuilder =
|
|
|
- new BalancerParameters.Builder();
|
|
|
- if (nodes != null) {
|
|
|
- pBuilder.setExcludedNodes(nodes.getNodesToBeExcluded());
|
|
|
- pBuilder.setIncludedNodes(nodes.getNodesToBeIncluded());
|
|
|
- pBuilder.setRunDuringUpgrade(false);
|
|
|
+ if(p == null) {
|
|
|
+ // run balancer and validate results
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ if (nodes != null) {
|
|
|
+ pBuilder.setExcludedNodes(nodes.getNodesToBeExcluded());
|
|
|
+ pBuilder.setIncludedNodes(nodes.getNodesToBeIncluded());
|
|
|
+ pBuilder.setRunDuringUpgrade(false);
|
|
|
+ }
|
|
|
+ p = pBuilder.build();
|
|
|
}
|
|
|
- BalancerParameters p = pBuilder.build();
|
|
|
-
|
|
|
int expectedExcludedNodes = 0;
|
|
|
if (nodes != null) {
|
|
|
if (!nodes.getNodesToBeExcluded().isEmpty()) {
|
|
@@ -821,6 +897,9 @@ public class TestBalancer {
|
|
|
== 0) {
|
|
|
assertEquals(ExitStatus.NO_MOVE_PROGRESS.getExitCode(), run);
|
|
|
return;
|
|
|
+ } else if(run == ExitStatus.NO_MOVE_BLOCK.getExitCode()) {
|
|
|
+ LOG.error("Exit status returned: " + run);
|
|
|
+ throw new Exception(String.valueOf(ExitStatus.NO_MOVE_BLOCK.getExitCode()));
|
|
|
} else {
|
|
|
assertEquals(ExitStatus.SUCCESS.getExitCode(), run);
|
|
|
}
|
|
@@ -828,8 +907,22 @@ public class TestBalancer {
|
|
|
LOG.info(" .");
|
|
|
try {
|
|
|
long totalUsedSpace = totalDfsUsedSpace + totalNonDfsUsedSpace;
|
|
|
- waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p,
|
|
|
- excludedNodes, checkExcludeNodesUtilization);
|
|
|
+ int expectedExcludedSourceNodes = 0;
|
|
|
+ int expectedExcludedTargetNodes = 0;
|
|
|
+ if(!p.getExcludedSourceNodes().isEmpty()) {
|
|
|
+ expectedExcludedSourceNodes = p.getExcludedSourceNodes().size();
|
|
|
+ }
|
|
|
+ if(!p.getExcludedTargetNodes().isEmpty()) {
|
|
|
+ expectedExcludedTargetNodes = p.getExcludedTargetNodes().size();
|
|
|
+ }
|
|
|
+ if(expectedExcludedSourceNodes > 0 || expectedExcludedTargetNodes > 0) {
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p,
|
|
|
+ expectedExcludedSourceNodes, expectedExcludedTargetNodes);
|
|
|
+ } else {
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p,
|
|
|
+ excludedNodes,
|
|
|
+ checkExcludeNodesUtilization);
|
|
|
+ }
|
|
|
} catch (TimeoutException e) {
|
|
|
// See HDFS-11682. NN may not get heartbeat to reflect the newest
|
|
|
// block changes.
|
|
@@ -879,8 +972,9 @@ public class TestBalancer {
|
|
|
b.resetData(conf);
|
|
|
if (r.getExitStatus() == ExitStatus.IN_PROGRESS) {
|
|
|
done = false;
|
|
|
- } else if (r.getExitStatus() != ExitStatus.SUCCESS) {
|
|
|
- //must be an error statue, return.
|
|
|
+ } else if (r.getExitStatus() != ExitStatus.SUCCESS
|
|
|
+ || r.getExitStatus() != ExitStatus.NO_MOVE_BLOCK) {
|
|
|
+ //must be an error status, return.
|
|
|
return r.getExitStatus().getExitCode();
|
|
|
} else {
|
|
|
if (iteration > 0) {
|
|
@@ -1129,7 +1223,7 @@ public class TestBalancer {
|
|
|
Configuration conf = new HdfsConfiguration();
|
|
|
initConf(conf);
|
|
|
doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
- CAPACITY, 1000L, RACK2, null, false, false, false, 0.3);
|
|
|
+ CAPACITY, 1000L, RACK2, null, false, false, false, 0.3, null);
|
|
|
}
|
|
|
|
|
|
private void testBalancerDefaultConstructor(Configuration conf,
|
|
@@ -1245,8 +1339,49 @@ public class TestBalancer {
|
|
|
Balancer.Cli.parse(parameters);
|
|
|
fail(reason + " for -source parameter");
|
|
|
} catch (IllegalArgumentException ignored) {
|
|
|
- // expected
|
|
|
+
|
|
|
+ }
|
|
|
+
|
|
|
+ parameters = new String[] {"-excludeSource"};
|
|
|
+ try {
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
+ fail(reason + " for -excludeSource parameter");
|
|
|
+ } catch (IllegalArgumentException ignored) {
|
|
|
+
|
|
|
+ }
|
|
|
+
|
|
|
+ parameters = new String[] {"-source", "testnode1", "-excludeSource", "testnode2"};
|
|
|
+ try {
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
+ fail("Exception is expected when both -source and -excludeSource are specified");
|
|
|
+ } catch (IllegalArgumentException e) {
|
|
|
+
|
|
|
+ }
|
|
|
+
|
|
|
+ parameters = new String[] {"-target"};
|
|
|
+ try {
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
+ fail(reason + " for -target parameter");
|
|
|
+ } catch (IllegalArgumentException ignored) {
|
|
|
+
|
|
|
}
|
|
|
+
|
|
|
+ parameters = new String[] {"-excludeTarget"};
|
|
|
+ try {
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
+ fail(reason + " for -excludeTarget parameter");
|
|
|
+ } catch (IllegalArgumentException ignored) {
|
|
|
+
|
|
|
+ }
|
|
|
+
|
|
|
+ parameters = new String[] {"-target", "testnode1", "-excludeTarget", "testnode2"};
|
|
|
+ try {
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
+ fail("Exception expected when both -target and -excludeTarget are specified");
|
|
|
+ } catch (IllegalArgumentException e) {
|
|
|
+
|
|
|
+ }
|
|
|
+
|
|
|
}
|
|
|
|
|
|
@Test
|
|
@@ -1920,7 +2055,7 @@ public class TestBalancer {
|
|
|
// all get block calls, so if two iterations are performed, the duration
|
|
|
// also includes the time it took to perform the block move ops in the
|
|
|
// first iteration
|
|
|
- new PortNumberBasedNodes(1, 0, 0), false, false, true, 0.5);
|
|
|
+ new PortNumberBasedNodes(1, 0, 0), false, false, true, 0.5, null);
|
|
|
assertTrue("Number of getBlocks should be not less than " +
|
|
|
getBlocksMaxQps, numGetBlocksCalls.get() >= getBlocksMaxQps);
|
|
|
long durationMs = 1 + endGetBlocksTime.get() - startGetBlocksTime.get();
|
|
@@ -1932,6 +2067,132 @@ public class TestBalancer {
|
|
|
getBlocksMaxQps, getBlockCallsPerSecond <= getBlocksMaxQps);
|
|
|
}
|
|
|
|
|
|
+ /**
|
|
|
+ * Test balancer with excluded target nodes.
|
|
|
+ * One of three added nodes is excluded in the target nodes list.
|
|
|
+ * Balancer should only move blocks to the two included nodes.
|
|
|
+ */
|
|
|
+ @Test(timeout=100000)
|
|
|
+ public void testBalancerExcludeTargetNodesNoMoveBlock() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ Set<String> excludeTargetNodes = new HashSet<>();
|
|
|
+ excludeTargetNodes.add("datanodeZ");
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ pBuilder.setExcludedTargetNodes(excludeTargetNodes);
|
|
|
+ BalancerParameters p = pBuilder.build();
|
|
|
+ Exception exception = assertThrows(Exception.class, () -> {
|
|
|
+ doTest(conf, CAPACITY, RACK2,
|
|
|
+ new HostNameBasedNodes(new String[]{"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
+ BalancerParameters.DEFAULT.getExcludedNodes(),
|
|
|
+ BalancerParameters.DEFAULT.getIncludedNodes()),
|
|
|
+ false, false, p);
|
|
|
+ });
|
|
|
+
|
|
|
+ assertTrue(exception.getMessage()
|
|
|
+ .contains(String.valueOf(ExitStatus.NO_MOVE_BLOCK.getExitCode())));
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Test balancer with included target nodes.
|
|
|
+ * Two of three added nodes are included in the target nodes list.
|
|
|
+ * Balancer should only move blocks to the included nodes.
|
|
|
+ */
|
|
|
+ @Test(timeout=100000)
|
|
|
+ public void testBalancerIncludeTargetNodesNoMoveBlock() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ Set<String> includeTargetNodes = new HashSet<>();
|
|
|
+ includeTargetNodes.add("datanodeY");
|
|
|
+ includeTargetNodes.add("datanodeZ");
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ pBuilder.setTargetNodes(includeTargetNodes);
|
|
|
+ BalancerParameters p = pBuilder.build();
|
|
|
+ Exception exception = assertThrows(Exception.class, () -> {
|
|
|
+ doTest(conf, CAPACITY, RACK2,
|
|
|
+ new HostNameBasedNodes(new String[]{"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
+ BalancerParameters.DEFAULT.getExcludedNodes(),
|
|
|
+ BalancerParameters.DEFAULT.getIncludedNodes()),
|
|
|
+ false, false, p);
|
|
|
+ });
|
|
|
+
|
|
|
+ assertTrue(exception.getMessage()
|
|
|
+ .contains(String.valueOf(ExitStatus.NO_MOVE_BLOCK.getExitCode())));
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Test balancer with included target nodes.
|
|
|
+ * Three of three added nodes are included in the target nodes list.
|
|
|
+ * Balancer should exit with success code.
|
|
|
+ */
|
|
|
+ @Test(timeout=100000)
|
|
|
+ public void testBalancerIncludeTargetNodesSuccess() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ Set<String> includeTargetNodes = new HashSet<>();
|
|
|
+ includeTargetNodes.add("datanodeX");
|
|
|
+ includeTargetNodes.add("datanodeY");
|
|
|
+ includeTargetNodes.add("datanodeZ");
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ pBuilder.setTargetNodes(includeTargetNodes);
|
|
|
+ BalancerParameters p = pBuilder.build();
|
|
|
+ doTest(conf, CAPACITY, RACK2,
|
|
|
+ new HostNameBasedNodes(new String[]{"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
+ BalancerParameters.DEFAULT.getExcludedNodes(),
|
|
|
+ BalancerParameters.DEFAULT.getIncludedNodes()),
|
|
|
+ false, false, p);
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Test balancer with included source nodes.
|
|
|
+ * Since newly added nodes are the only included source nodes no balancing will occur.
|
|
|
+ */
|
|
|
+ @Test(timeout=100000)
|
|
|
+ public void testBalancerIncludeSourceNodesNoMoveBlock() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ Set<String> includeSourceNodes = new HashSet<>();
|
|
|
+ includeSourceNodes.add("datanodeX");
|
|
|
+ includeSourceNodes.add("datanodeY");
|
|
|
+ includeSourceNodes.add("datanodeZ");
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ pBuilder.setSourceNodes(includeSourceNodes);
|
|
|
+ BalancerParameters p = pBuilder.build();
|
|
|
+ Exception exception = assertThrows(Exception.class, () -> {
|
|
|
+ doTest(conf, CAPACITY, RACK2,
|
|
|
+ new HostNameBasedNodes(new String[]{"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
+ BalancerParameters.DEFAULT.getExcludedNodes(),
|
|
|
+ BalancerParameters.DEFAULT.getIncludedNodes()),
|
|
|
+ false, false, p);
|
|
|
+ });
|
|
|
+
|
|
|
+ assertTrue(exception.getMessage()
|
|
|
+ .contains(String.valueOf(ExitStatus.NO_MOVE_BLOCK.getExitCode())));
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Test balancer with excluded source nodes.
|
|
|
+ * Since newly added nodes will not be selected as a source,
|
|
|
+ * all nodes will be included in balancing.
|
|
|
+ */
|
|
|
+ @Test(timeout=100000)
|
|
|
+ public void testBalancerExcludeSourceNodes() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ Set<String> excludeSourceNodes = new HashSet<>();
|
|
|
+ excludeSourceNodes.add("datanodeX");
|
|
|
+ excludeSourceNodes.add("datanodeY");
|
|
|
+ BalancerParameters.Builder pBuilder = new BalancerParameters.Builder();
|
|
|
+ pBuilder.setExcludedSourceNodes(excludeSourceNodes);
|
|
|
+ BalancerParameters p = pBuilder.build();
|
|
|
+ doTest(conf, CAPACITY, RACK2,
|
|
|
+ new HostNameBasedNodes(new String[]{"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
+ BalancerParameters.DEFAULT.getExcludedNodes(),
|
|
|
+ BalancerParameters.DEFAULT.getIncludedNodes()), false,
|
|
|
+ false, p);
|
|
|
+ }
|
|
|
+
|
|
|
+
|
|
|
/**
|
|
|
* @param args
|
|
|
*/
|