|
@@ -30,9 +30,13 @@ import javax.management.MBeanServer;
|
|
|
import javax.management.ObjectName;
|
|
|
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
+import org.apache.hadoop.fs.FileSystem;
|
|
|
import org.apache.hadoop.fs.FileUtil;
|
|
|
+import org.apache.hadoop.fs.Path;
|
|
|
import org.apache.hadoop.hdfs.DFSConfigKeys;
|
|
|
+import org.apache.hadoop.hdfs.DFSTestUtil;
|
|
|
import org.apache.hadoop.hdfs.MiniDFSCluster;
|
|
|
+import org.apache.hadoop.hdfs.server.datanode.DataNode;
|
|
|
import org.apache.hadoop.io.nativeio.NativeIO;
|
|
|
import org.apache.hadoop.io.nativeio.NativeIO.POSIX.NoMlockCacheManipulator;
|
|
|
import org.apache.hadoop.util.VersionInfo;
|
|
@@ -58,11 +62,14 @@ public class TestNameNodeMXBean {
|
|
|
public void testNameNodeMXBeanInfo() throws Exception {
|
|
|
Configuration conf = new Configuration();
|
|
|
conf.setLong(DFSConfigKeys.DFS_DATANODE_MAX_LOCKED_MEMORY_KEY,
|
|
|
- NativeIO.POSIX.getCacheManipulator().getMemlockLimit());
|
|
|
+ NativeIO.POSIX.getCacheManipulator().getMemlockLimit());
|
|
|
+ conf.setInt(DFSConfigKeys.DFS_HEARTBEAT_INTERVAL_KEY, 1);
|
|
|
+ conf.setInt(DFSConfigKeys.DFS_NAMENODE_HEARTBEAT_RECHECK_INTERVAL_KEY, 1);
|
|
|
+
|
|
|
MiniDFSCluster cluster = null;
|
|
|
|
|
|
try {
|
|
|
- cluster = new MiniDFSCluster.Builder(conf).build();
|
|
|
+ cluster = new MiniDFSCluster.Builder(conf).numDataNodes(3).build();
|
|
|
cluster.waitActive();
|
|
|
|
|
|
FSNamesystem fsn = cluster.getNameNode().namesystem;
|
|
@@ -70,6 +77,29 @@ public class TestNameNodeMXBean {
|
|
|
MBeanServer mbs = ManagementFactory.getPlatformMBeanServer();
|
|
|
ObjectName mxbeanName = new ObjectName(
|
|
|
"Hadoop:service=NameNode,name=NameNodeInfo");
|
|
|
+
|
|
|
+ // Define include file to generate deadNodes metrics
|
|
|
+ FileSystem localFileSys = FileSystem.getLocal(conf);
|
|
|
+ Path workingDir = localFileSys.getWorkingDirectory();
|
|
|
+ Path dir = new Path(workingDir,
|
|
|
+ "build/test/data/temp/TestNameNodeMXBean");
|
|
|
+ Path includeFile = new Path(dir, "include");
|
|
|
+ assertTrue(localFileSys.mkdirs(dir));
|
|
|
+ StringBuilder includeHosts = new StringBuilder();
|
|
|
+ for(DataNode dn : cluster.getDataNodes()) {
|
|
|
+ includeHosts.append(dn.getDisplayName()).append("\n");
|
|
|
+ }
|
|
|
+ DFSTestUtil.writeFile(localFileSys, includeFile, includeHosts.toString());
|
|
|
+ conf.set(DFSConfigKeys.DFS_HOSTS, includeFile.toUri().getPath());
|
|
|
+ fsn.getBlockManager().getDatanodeManager().refreshNodes(conf);
|
|
|
+
|
|
|
+ cluster.stopDataNode(0);
|
|
|
+ while (fsn.getNumDatanodesInService() != 2) {
|
|
|
+ try {
|
|
|
+ Thread.sleep(1000);
|
|
|
+ } catch (InterruptedException e) {}
|
|
|
+ }
|
|
|
+
|
|
|
// get attribute "ClusterId"
|
|
|
String clusterId = (String) mbs.getAttribute(mxbeanName, "ClusterId");
|
|
|
assertEquals(fsn.getClusterId(), clusterId);
|
|
@@ -121,6 +151,15 @@ public class TestNameNodeMXBean {
|
|
|
String deadnodeinfo = (String) (mbs.getAttribute(mxbeanName,
|
|
|
"DeadNodes"));
|
|
|
assertEquals(fsn.getDeadNodes(), deadnodeinfo);
|
|
|
+ Map<String, Map<String, Object>> deadNodes =
|
|
|
+ (Map<String, Map<String, Object>>) JSON.parse(deadnodeinfo);
|
|
|
+ assertTrue(deadNodes.size() > 0);
|
|
|
+ for (Map<String, Object> deadNode : deadNodes.values()) {
|
|
|
+ assertTrue(deadNode.containsKey("lastContact"));
|
|
|
+ assertTrue(deadNode.containsKey("decommissioned"));
|
|
|
+ assertTrue(deadNode.containsKey("xferaddr"));
|
|
|
+ }
|
|
|
+
|
|
|
// get attribute NodeUsage
|
|
|
String nodeUsage = (String) (mbs.getAttribute(mxbeanName,
|
|
|
"NodeUsage"));
|
|
@@ -181,7 +220,7 @@ public class TestNameNodeMXBean {
|
|
|
assertEquals(1, statusMap.get("active").size());
|
|
|
assertEquals(1, statusMap.get("failed").size());
|
|
|
assertEquals(0L, mbs.getAttribute(mxbeanName, "CacheUsed"));
|
|
|
- assertEquals(NativeIO.POSIX.getCacheManipulator().getMemlockLimit() *
|
|
|
+ assertEquals(NativeIO.POSIX.getCacheManipulator().getMemlockLimit() *
|
|
|
cluster.getDataNodes().size(),
|
|
|
mbs.getAttribute(mxbeanName, "CacheCapacity"));
|
|
|
} finally {
|