|
@@ -18,17 +18,23 @@
|
|
package org.apache.hadoop.hdfs.server.balancer;
|
|
package org.apache.hadoop.hdfs.server.balancer;
|
|
|
|
|
|
import static org.junit.Assert.assertEquals;
|
|
import static org.junit.Assert.assertEquals;
|
|
|
|
+import static org.junit.Assert.assertTrue;
|
|
import static org.junit.Assert.fail;
|
|
import static org.junit.Assert.fail;
|
|
|
|
|
|
|
|
+import java.io.File;
|
|
import java.io.IOException;
|
|
import java.io.IOException;
|
|
|
|
+import java.io.PrintWriter;
|
|
import java.net.URI;
|
|
import java.net.URI;
|
|
import java.util.ArrayList;
|
|
import java.util.ArrayList;
|
|
import java.util.Arrays;
|
|
import java.util.Arrays;
|
|
import java.util.Collection;
|
|
import java.util.Collection;
|
|
|
|
+import java.util.HashSet;
|
|
import java.util.List;
|
|
import java.util.List;
|
|
import java.util.Random;
|
|
import java.util.Random;
|
|
|
|
+import java.util.Set;
|
|
import java.util.concurrent.TimeoutException;
|
|
import java.util.concurrent.TimeoutException;
|
|
|
|
|
|
|
|
+import org.apache.commons.lang.StringUtils;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.impl.Log4JLogger;
|
|
import org.apache.commons.logging.impl.Log4JLogger;
|
|
@@ -48,6 +54,8 @@ import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
|
|
import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
|
|
import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
|
|
import org.apache.hadoop.hdfs.protocol.LocatedBlock;
|
|
import org.apache.hadoop.hdfs.protocol.LocatedBlock;
|
|
import org.apache.hadoop.hdfs.server.balancer.Balancer.Cli;
|
|
import org.apache.hadoop.hdfs.server.balancer.Balancer.Cli;
|
|
|
|
+import org.apache.hadoop.hdfs.server.balancer.Balancer.Parameters;
|
|
|
|
+import org.apache.hadoop.hdfs.server.datanode.DataNode;
|
|
import org.apache.hadoop.hdfs.server.datanode.SimulatedFSDataset;
|
|
import org.apache.hadoop.hdfs.server.datanode.SimulatedFSDataset;
|
|
import org.apache.hadoop.util.Time;
|
|
import org.apache.hadoop.util.Time;
|
|
import org.apache.hadoop.util.Tool;
|
|
import org.apache.hadoop.util.Tool;
|
|
@@ -255,6 +263,18 @@ public class TestBalancer {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Wait until balanced: each datanode gives utilization within
|
|
|
|
+ * BALANCE_ALLOWED_VARIANCE of average
|
|
|
|
+ * @throws IOException
|
|
|
|
+ * @throws TimeoutException
|
|
|
|
+ */
|
|
|
|
+ static void waitForBalancer(long totalUsedSpace, long totalCapacity,
|
|
|
|
+ ClientProtocol client, MiniDFSCluster cluster, Balancer.Parameters p)
|
|
|
|
+ throws IOException, TimeoutException {
|
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p, 0);
|
|
|
|
+ }
|
|
|
|
|
|
/**
|
|
/**
|
|
* Wait until balanced: each datanode gives utilization within
|
|
* Wait until balanced: each datanode gives utilization within
|
|
@@ -263,11 +283,17 @@ public class TestBalancer {
|
|
* @throws TimeoutException
|
|
* @throws TimeoutException
|
|
*/
|
|
*/
|
|
static void waitForBalancer(long totalUsedSpace, long totalCapacity,
|
|
static void waitForBalancer(long totalUsedSpace, long totalCapacity,
|
|
- ClientProtocol client, MiniDFSCluster cluster)
|
|
|
|
- throws IOException, TimeoutException {
|
|
|
|
|
|
+ ClientProtocol client, MiniDFSCluster cluster, Balancer.Parameters p,
|
|
|
|
+ int expectedExcludedNodes) throws IOException, TimeoutException {
|
|
long timeout = TIMEOUT;
|
|
long timeout = TIMEOUT;
|
|
long failtime = (timeout <= 0L) ? Long.MAX_VALUE
|
|
long failtime = (timeout <= 0L) ? Long.MAX_VALUE
|
|
: Time.now() + timeout;
|
|
: Time.now() + timeout;
|
|
|
|
+ if (!p.nodesToBeIncluded.isEmpty()) {
|
|
|
|
+ totalCapacity = p.nodesToBeIncluded.size() * CAPACITY;
|
|
|
|
+ }
|
|
|
|
+ if (!p.nodesToBeExcluded.isEmpty()) {
|
|
|
|
+ totalCapacity -= p.nodesToBeExcluded.size() * CAPACITY;
|
|
|
|
+ }
|
|
final double avgUtilization = ((double)totalUsedSpace) / totalCapacity;
|
|
final double avgUtilization = ((double)totalUsedSpace) / totalCapacity;
|
|
boolean balanced;
|
|
boolean balanced;
|
|
do {
|
|
do {
|
|
@@ -275,9 +301,20 @@ public class TestBalancer {
|
|
client.getDatanodeReport(DatanodeReportType.ALL);
|
|
client.getDatanodeReport(DatanodeReportType.ALL);
|
|
assertEquals(datanodeReport.length, cluster.getDataNodes().size());
|
|
assertEquals(datanodeReport.length, cluster.getDataNodes().size());
|
|
balanced = true;
|
|
balanced = true;
|
|
|
|
+ int actualExcludedNodeCount = 0;
|
|
for (DatanodeInfo datanode : datanodeReport) {
|
|
for (DatanodeInfo datanode : datanodeReport) {
|
|
double nodeUtilization = ((double)datanode.getDfsUsed())
|
|
double nodeUtilization = ((double)datanode.getDfsUsed())
|
|
/ datanode.getCapacity();
|
|
/ datanode.getCapacity();
|
|
|
|
+ if (Balancer.Util.shouldBeExcluded(p.nodesToBeExcluded, datanode)) {
|
|
|
|
+ assertTrue(nodeUtilization == 0);
|
|
|
|
+ actualExcludedNodeCount++;
|
|
|
|
+ continue;
|
|
|
|
+ }
|
|
|
|
+ if (!Balancer.Util.shouldBeIncluded(p.nodesToBeIncluded, datanode)) {
|
|
|
|
+ assertTrue(nodeUtilization == 0);
|
|
|
|
+ actualExcludedNodeCount++;
|
|
|
|
+ continue;
|
|
|
|
+ }
|
|
if (Math.abs(avgUtilization - nodeUtilization) > BALANCE_ALLOWED_VARIANCE) {
|
|
if (Math.abs(avgUtilization - nodeUtilization) > BALANCE_ALLOWED_VARIANCE) {
|
|
balanced = false;
|
|
balanced = false;
|
|
if (Time.now() > failtime) {
|
|
if (Time.now() > failtime) {
|
|
@@ -294,6 +331,7 @@ public class TestBalancer {
|
|
break;
|
|
break;
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
+ assertEquals(expectedExcludedNodes,actualExcludedNodeCount);
|
|
} while (!balanced);
|
|
} while (!balanced);
|
|
}
|
|
}
|
|
|
|
|
|
@@ -307,22 +345,118 @@ public class TestBalancer {
|
|
}
|
|
}
|
|
return b.append("]").toString();
|
|
return b.append("]").toString();
|
|
}
|
|
}
|
|
- /** This test start a cluster with specified number of nodes,
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Class which contains information about the
|
|
|
|
+ * new nodes to be added to the cluster for balancing.
|
|
|
|
+ */
|
|
|
|
+ static abstract class NewNodeInfo {
|
|
|
|
+
|
|
|
|
+ Set<String> nodesToBeExcluded = new HashSet<String>();
|
|
|
|
+ Set<String> nodesToBeIncluded = new HashSet<String>();
|
|
|
|
+
|
|
|
|
+ abstract String[] getNames();
|
|
|
|
+ abstract int getNumberofNewNodes();
|
|
|
|
+ abstract int getNumberofIncludeNodes();
|
|
|
|
+ abstract int getNumberofExcludeNodes();
|
|
|
|
+
|
|
|
|
+ public Set<String> getNodesToBeIncluded() {
|
|
|
|
+ return nodesToBeIncluded;
|
|
|
|
+ }
|
|
|
|
+ public Set<String> getNodesToBeExcluded() {
|
|
|
|
+ return nodesToBeExcluded;
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * The host names of new nodes are specified
|
|
|
|
+ */
|
|
|
|
+ static class HostNameBasedNodes extends NewNodeInfo {
|
|
|
|
+ String[] hostnames;
|
|
|
|
+
|
|
|
|
+ public HostNameBasedNodes(String[] hostnames,
|
|
|
|
+ Set<String> nodesToBeExcluded, Set<String> nodesToBeIncluded) {
|
|
|
|
+ this.hostnames = hostnames;
|
|
|
|
+ this.nodesToBeExcluded = nodesToBeExcluded;
|
|
|
|
+ this.nodesToBeIncluded = nodesToBeIncluded;
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Override
|
|
|
|
+ String[] getNames() {
|
|
|
|
+ return hostnames;
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofNewNodes() {
|
|
|
|
+ return hostnames.length;
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofIncludeNodes() {
|
|
|
|
+ return nodesToBeIncluded.size();
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofExcludeNodes() {
|
|
|
|
+ return nodesToBeExcluded.size();
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * The number of data nodes to be started are specified.
|
|
|
|
+ * The data nodes will have same host name, but different port numbers.
|
|
|
|
+ *
|
|
|
|
+ */
|
|
|
|
+ static class PortNumberBasedNodes extends NewNodeInfo {
|
|
|
|
+ int newNodes;
|
|
|
|
+ int excludeNodes;
|
|
|
|
+ int includeNodes;
|
|
|
|
+
|
|
|
|
+ public PortNumberBasedNodes(int newNodes, int excludeNodes, int includeNodes) {
|
|
|
|
+ this.newNodes = newNodes;
|
|
|
|
+ this.excludeNodes = excludeNodes;
|
|
|
|
+ this.includeNodes = includeNodes;
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Override
|
|
|
|
+ String[] getNames() {
|
|
|
|
+ return null;
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofNewNodes() {
|
|
|
|
+ return newNodes;
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofIncludeNodes() {
|
|
|
|
+ return includeNodes;
|
|
|
|
+ }
|
|
|
|
+ @Override
|
|
|
|
+ int getNumberofExcludeNodes() {
|
|
|
|
+ return excludeNodes;
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ 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);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /** This test start a cluster with specified number of nodes,
|
|
* and fills it to be 30% full (with a single file replicated identically
|
|
* and fills it to be 30% full (with a single file replicated identically
|
|
* to all datanodes);
|
|
* to all datanodes);
|
|
* It then adds one new empty node and starts balancing.
|
|
* It then adds one new empty node and starts balancing.
|
|
- *
|
|
|
|
|
|
+ *
|
|
* @param conf - configuration
|
|
* @param conf - configuration
|
|
* @param capacities - array of capacities of original nodes in cluster
|
|
* @param capacities - array of capacities of original nodes in cluster
|
|
* @param racks - array of racks for original nodes in cluster
|
|
* @param racks - array of racks for original nodes in cluster
|
|
* @param newCapacity - new node's capacity
|
|
* @param newCapacity - new node's capacity
|
|
* @param newRack - new node's rack
|
|
* @param newRack - new node's rack
|
|
|
|
+ * @param nodes - information about new nodes to be started.
|
|
* @param useTool - if true run test via Cli with command-line argument
|
|
* @param useTool - if true run test via Cli with command-line argument
|
|
* parsing, etc. Otherwise invoke balancer API directly.
|
|
* parsing, etc. Otherwise invoke balancer API directly.
|
|
|
|
+ * @param useFile - if true, the hosts to included or excluded will be stored in a
|
|
|
|
+ * file and then later read from the file.
|
|
* @throws Exception
|
|
* @throws Exception
|
|
*/
|
|
*/
|
|
- private void doTest(Configuration conf, long[] capacities, String[] racks,
|
|
|
|
- long newCapacity, String newRack, boolean useTool) throws Exception {
|
|
|
|
|
|
+ private void doTest(Configuration conf, long[] capacities,
|
|
|
|
+ String[] racks, long newCapacity, String newRack, NewNodeInfo nodes,
|
|
|
|
+ boolean useTool, boolean useFile) throws Exception {
|
|
LOG.info("capacities = " + long2String(capacities));
|
|
LOG.info("capacities = " + long2String(capacities));
|
|
LOG.info("racks = " + Arrays.asList(racks));
|
|
LOG.info("racks = " + Arrays.asList(racks));
|
|
LOG.info("newCapacity= " + newCapacity);
|
|
LOG.info("newCapacity= " + newCapacity);
|
|
@@ -346,17 +480,75 @@ public class TestBalancer {
|
|
long totalUsedSpace = totalCapacity*3/10;
|
|
long totalUsedSpace = totalCapacity*3/10;
|
|
createFile(cluster, filePath, totalUsedSpace / numOfDatanodes,
|
|
createFile(cluster, filePath, totalUsedSpace / numOfDatanodes,
|
|
(short) numOfDatanodes, 0);
|
|
(short) numOfDatanodes, 0);
|
|
- // start up an empty node with the same capacity and on the same rack
|
|
|
|
- cluster.startDataNodes(conf, 1, true, null,
|
|
|
|
- new String[]{newRack}, new long[]{newCapacity});
|
|
|
|
|
|
|
|
- totalCapacity += newCapacity;
|
|
|
|
|
|
+ if (nodes == null) { // there is no specification of new nodes.
|
|
|
|
+ // start up an empty node with the same capacity and on the same rack
|
|
|
|
+ cluster.startDataNodes(conf, 1, true, null,
|
|
|
|
+ new String[]{newRack}, null,new long[]{newCapacity});
|
|
|
|
+ totalCapacity += newCapacity;
|
|
|
|
+ } else {
|
|
|
|
+ //if running a test with "include list", include original nodes as well
|
|
|
|
+ if (nodes.getNumberofIncludeNodes()>0) {
|
|
|
|
+ for (DataNode dn: cluster.getDataNodes())
|
|
|
|
+ nodes.getNodesToBeIncluded().add(dn.getDatanodeId().getHostName());
|
|
|
|
+ }
|
|
|
|
+ String[] newRacks = new String[nodes.getNumberofNewNodes()];
|
|
|
|
+ long[] newCapacities = new long[nodes.getNumberofNewNodes()];
|
|
|
|
+ for (int i=0; i < nodes.getNumberofNewNodes(); i++) {
|
|
|
|
+ newRacks[i] = newRack;
|
|
|
|
+ newCapacities[i] = newCapacity;
|
|
|
|
+ }
|
|
|
|
+ // if host names are specified for the new nodes to be created.
|
|
|
|
+ if (nodes.getNames() != null) {
|
|
|
|
+ cluster.startDataNodes(conf, nodes.getNumberofNewNodes(), true, null,
|
|
|
|
+ newRacks, nodes.getNames(), newCapacities);
|
|
|
|
+ totalCapacity += newCapacity*nodes.getNumberofNewNodes();
|
|
|
|
+ } else { // host names are not specified
|
|
|
|
+ cluster.startDataNodes(conf, nodes.getNumberofNewNodes(), true, null,
|
|
|
|
+ newRacks, null, newCapacities);
|
|
|
|
+ totalCapacity += newCapacity*nodes.getNumberofNewNodes();
|
|
|
|
+ //populate the include nodes
|
|
|
|
+ if (nodes.getNumberofIncludeNodes() > 0) {
|
|
|
|
+ int totalNodes = cluster.getDataNodes().size();
|
|
|
|
+ for (int i=0; i < nodes.getNumberofIncludeNodes(); i++) {
|
|
|
|
+ nodes.getNodesToBeIncluded().add (cluster.getDataNodes().get(
|
|
|
|
+ totalNodes-1-i).getDatanodeId().getXferAddr());
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ //polulate the exclude nodes
|
|
|
|
+ if (nodes.getNumberofExcludeNodes() > 0) {
|
|
|
|
+ int totalNodes = cluster.getDataNodes().size();
|
|
|
|
+ for (int i=0; i < nodes.getNumberofExcludeNodes(); i++) {
|
|
|
|
+ nodes.getNodesToBeExcluded().add (cluster.getDataNodes().get(
|
|
|
|
+ totalNodes-1-i).getDatanodeId().getXferAddr());
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ // run balancer and validate results
|
|
|
|
+ Balancer.Parameters p = Balancer.Parameters.DEFAULT;
|
|
|
|
+ if (nodes != null) {
|
|
|
|
+ p = new Balancer.Parameters(
|
|
|
|
+ Balancer.Parameters.DEFAULT.policy,
|
|
|
|
+ Balancer.Parameters.DEFAULT.threshold,
|
|
|
|
+ nodes.getNodesToBeExcluded(), nodes.getNodesToBeIncluded());
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ int expectedExcludedNodes = 0;
|
|
|
|
+ if (nodes != null) {
|
|
|
|
+ if (!nodes.getNodesToBeExcluded().isEmpty()) {
|
|
|
|
+ expectedExcludedNodes = nodes.getNodesToBeExcluded().size();
|
|
|
|
+ } else if (!nodes.getNodesToBeIncluded().isEmpty()) {
|
|
|
|
+ expectedExcludedNodes =
|
|
|
|
+ cluster.getDataNodes().size() - nodes.getNodesToBeIncluded().size();
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
|
|
// run balancer and validate results
|
|
// run balancer and validate results
|
|
if (useTool) {
|
|
if (useTool) {
|
|
- runBalancerCli(conf, totalUsedSpace, totalCapacity);
|
|
|
|
|
|
+ runBalancerCli(conf, totalUsedSpace, totalCapacity, p, useFile, expectedExcludedNodes);
|
|
} else {
|
|
} else {
|
|
- runBalancer(conf, totalUsedSpace, totalCapacity);
|
|
|
|
|
|
+ runBalancer(conf, totalUsedSpace, totalCapacity, p, expectedExcludedNodes);
|
|
}
|
|
}
|
|
} finally {
|
|
} finally {
|
|
cluster.shutdown();
|
|
cluster.shutdown();
|
|
@@ -365,11 +557,17 @@ public class TestBalancer {
|
|
|
|
|
|
private void runBalancer(Configuration conf,
|
|
private void runBalancer(Configuration conf,
|
|
long totalUsedSpace, long totalCapacity) throws Exception {
|
|
long totalUsedSpace, long totalCapacity) throws Exception {
|
|
|
|
+ runBalancer(conf, totalUsedSpace, totalCapacity, Balancer.Parameters.DEFAULT, 0);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private void runBalancer(Configuration conf,
|
|
|
|
+ long totalUsedSpace, long totalCapacity, Balancer.Parameters p,
|
|
|
|
+ int excludedNodes) throws Exception {
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
|
|
|
|
// start rebalancing
|
|
// start rebalancing
|
|
Collection<URI> namenodes = DFSUtil.getNsServiceRpcUris(conf);
|
|
Collection<URI> namenodes = DFSUtil.getNsServiceRpcUris(conf);
|
|
- final int r = Balancer.run(namenodes, Balancer.Parameters.DEFALUT, conf);
|
|
|
|
|
|
+ final int r = Balancer.run(namenodes, p, conf);
|
|
if (conf.getInt(DFSConfigKeys.DFS_DATANODE_BALANCE_MAX_NUM_CONCURRENT_MOVES_KEY,
|
|
if (conf.getInt(DFSConfigKeys.DFS_DATANODE_BALANCE_MAX_NUM_CONCURRENT_MOVES_KEY,
|
|
DFSConfigKeys.DFS_DATANODE_BALANCE_MAX_NUM_CONCURRENT_MOVES_DEFAULT) ==0) {
|
|
DFSConfigKeys.DFS_DATANODE_BALANCE_MAX_NUM_CONCURRENT_MOVES_DEFAULT) ==0) {
|
|
assertEquals(Balancer.ReturnStatus.NO_MOVE_PROGRESS.code, r);
|
|
assertEquals(Balancer.ReturnStatus.NO_MOVE_PROGRESS.code, r);
|
|
@@ -379,22 +577,66 @@ public class TestBalancer {
|
|
}
|
|
}
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
LOG.info("Rebalancing with default ctor.");
|
|
LOG.info("Rebalancing with default ctor.");
|
|
- waitForBalancer(totalUsedSpace, totalCapacity, client, cluster);
|
|
|
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p, excludedNodes);
|
|
}
|
|
}
|
|
-
|
|
|
|
|
|
+
|
|
private void runBalancerCli(Configuration conf,
|
|
private void runBalancerCli(Configuration conf,
|
|
- long totalUsedSpace, long totalCapacity) throws Exception {
|
|
|
|
|
|
+ long totalUsedSpace, long totalCapacity,
|
|
|
|
+ Balancer.Parameters p, boolean useFile, int expectedExcludedNodes) throws Exception {
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
|
|
+ List <String> args = new ArrayList<String>();
|
|
|
|
+ args.add("-policy");
|
|
|
|
+ args.add("datanode");
|
|
|
|
+
|
|
|
|
+ File excludeHostsFile = null;
|
|
|
|
+ if (!p.nodesToBeExcluded.isEmpty()) {
|
|
|
|
+ args.add("-exclude");
|
|
|
|
+ if (useFile) {
|
|
|
|
+ excludeHostsFile = new File ("exclude-hosts-file");
|
|
|
|
+ PrintWriter pw = new PrintWriter(excludeHostsFile);
|
|
|
|
+ for (String host: p.nodesToBeExcluded) {
|
|
|
|
+ pw.write( host + "\n");
|
|
|
|
+ }
|
|
|
|
+ pw.close();
|
|
|
|
+ args.add("-f");
|
|
|
|
+ args.add("exclude-hosts-file");
|
|
|
|
+ } else {
|
|
|
|
+ args.add(StringUtils.join(p.nodesToBeExcluded, ','));
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ File includeHostsFile = null;
|
|
|
|
+ if (!p.nodesToBeIncluded.isEmpty()) {
|
|
|
|
+ args.add("-include");
|
|
|
|
+ if (useFile) {
|
|
|
|
+ includeHostsFile = new File ("include-hosts-file");
|
|
|
|
+ PrintWriter pw = new PrintWriter(includeHostsFile);
|
|
|
|
+ for (String host: p.nodesToBeIncluded){
|
|
|
|
+ pw.write( host + "\n");
|
|
|
|
+ }
|
|
|
|
+ pw.close();
|
|
|
|
+ args.add("-f");
|
|
|
|
+ args.add("include-hosts-file");
|
|
|
|
+ } else {
|
|
|
|
+ args.add(StringUtils.join(p.nodesToBeIncluded, ','));
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
|
|
- final String[] args = { "-policy", "datanode" };
|
|
|
|
final Tool tool = new Cli();
|
|
final Tool tool = new Cli();
|
|
tool.setConf(conf);
|
|
tool.setConf(conf);
|
|
- final int r = tool.run(args); // start rebalancing
|
|
|
|
|
|
+ final int r = tool.run(args.toArray(new String[0])); // start rebalancing
|
|
|
|
|
|
assertEquals("Tools should exit 0 on success", 0, r);
|
|
assertEquals("Tools should exit 0 on success", 0, r);
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
waitForHeartBeat(totalUsedSpace, totalCapacity, client, cluster);
|
|
LOG.info("Rebalancing with default ctor.");
|
|
LOG.info("Rebalancing with default ctor.");
|
|
- waitForBalancer(totalUsedSpace, totalCapacity, client, cluster);
|
|
|
|
|
|
+ waitForBalancer(totalUsedSpace, totalCapacity, client, cluster, p, expectedExcludedNodes);
|
|
|
|
+
|
|
|
|
+ if (excludeHostsFile != null && excludeHostsFile.exists()) {
|
|
|
|
+ excludeHostsFile.delete();
|
|
|
|
+ }
|
|
|
|
+ if (includeHostsFile != null && includeHostsFile.exists()) {
|
|
|
|
+ includeHostsFile.delete();
|
|
|
|
+ }
|
|
}
|
|
}
|
|
|
|
|
|
/** one-node cluster test*/
|
|
/** one-node cluster test*/
|
|
@@ -440,7 +682,7 @@ public class TestBalancer {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- /** Test a cluster with even distribution,
|
|
|
|
|
|
+ /** Test a cluster with even distribution,
|
|
* then a new empty node is added to the cluster*/
|
|
* then a new empty node is added to the cluster*/
|
|
@Test(timeout=100000)
|
|
@Test(timeout=100000)
|
|
public void testBalancer0() throws Exception {
|
|
public void testBalancer0() throws Exception {
|
|
@@ -554,7 +796,13 @@ public class TestBalancer {
|
|
} catch (IllegalArgumentException e) {
|
|
} catch (IllegalArgumentException e) {
|
|
|
|
|
|
}
|
|
}
|
|
|
|
+ parameters = new String[] {"-include", "testnode1", "-exclude", "testnode2"};
|
|
|
|
+ try {
|
|
|
|
+ Balancer.Cli.parse(parameters);
|
|
|
|
+ fail("IllegalArgumentException is expected when both -exclude and -include are specified");
|
|
|
|
+ } catch (IllegalArgumentException e) {
|
|
|
|
|
|
|
|
+ }
|
|
}
|
|
}
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -569,6 +817,183 @@ public class TestBalancer {
|
|
oneNodeTest(conf, true);
|
|
oneNodeTest(conf, true);
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerWithExcludeList() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> excludeHosts = new HashSet<String>();
|
|
|
|
+ excludeHosts.add( "datanodeY");
|
|
|
|
+ excludeHosts.add( "datanodeZ");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
|
+ excludeHosts, Parameters.DEFAULT.nodesToBeIncluded), false, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerWithExcludeListWithPorts() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 2, 0), false, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithExcludeList() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> excludeHosts = new HashSet<String>();
|
|
|
|
+ excludeHosts.add( "datanodeY");
|
|
|
|
+ excludeHosts.add( "datanodeZ");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"}, excludeHosts,
|
|
|
|
+ Parameters.DEFAULT.nodesToBeIncluded), true, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithExcludeListWithPorts() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 2, 0), true, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list in a file
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithExcludeListInAFile() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> excludeHosts = new HashSet<String>();
|
|
|
|
+ excludeHosts.add( "datanodeY");
|
|
|
|
+ excludeHosts.add( "datanodeZ");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
|
+ excludeHosts, Parameters.DEFAULT.nodesToBeIncluded), true, true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,G
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the exclude list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithExcludeListWithPortsInAFile() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 2, 0), true, true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerWithIncludeList() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> includeHosts = new HashSet<String>();
|
|
|
|
+ includeHosts.add( "datanodeY");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
|
+ Parameters.DEFAULT.nodesToBeExcluded, includeHosts), false, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerWithIncludeListWithPorts() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 0, 1), false, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithIncludeList() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> includeHosts = new HashSet<String>();
|
|
|
|
+ includeHosts.add( "datanodeY");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
|
+ Parameters.DEFAULT.nodesToBeExcluded, includeHosts), true, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithIncludeListWithPorts() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 0, 1), true, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithIncludeListInAFile() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ Set<String> includeHosts = new HashSet<String>();
|
|
|
|
+ includeHosts.add( "datanodeY");
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1}, CAPACITY, RACK2,
|
|
|
|
+ new HostNameBasedNodes(new String[] {"datanodeX", "datanodeY", "datanodeZ"},
|
|
|
|
+ Parameters.DEFAULT.nodesToBeExcluded, includeHosts), true, true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Test a cluster with even distribution,
|
|
|
|
+ * then three nodes are added to the cluster,
|
|
|
|
+ * runs balancer with two of the nodes in the include list
|
|
|
|
+ */
|
|
|
|
+ @Test(timeout=100000)
|
|
|
|
+ public void testBalancerCliWithIncludeListWithPortsInAFile() throws Exception {
|
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
|
+ initConf(conf);
|
|
|
|
+ doTest(conf, new long[]{CAPACITY, CAPACITY}, new String[]{RACK0, RACK1},
|
|
|
|
+ CAPACITY, RACK2, new PortNumberBasedNodes(3, 0, 1), true, true);
|
|
|
|
+ }
|
|
|
|
+
|
|
/**
|
|
/**
|
|
* @param args
|
|
* @param args
|
|
*/
|
|
*/
|