|
@@ -21,6 +21,9 @@ import com.google.common.annotations.VisibleForTesting;
|
|
|
import com.google.common.base.Joiner;
|
|
|
import com.google.common.base.Preconditions;
|
|
|
import com.google.common.collect.Lists;
|
|
|
+import org.apache.commons.logging.Log;
|
|
|
+import org.apache.commons.logging.LogFactory;
|
|
|
+import org.apache.commons.logging.impl.Log4JLogger;
|
|
|
import org.apache.hadoop.HadoopIllegalArgumentException;
|
|
|
import org.apache.hadoop.classification.InterfaceAudience;
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
@@ -74,20 +77,34 @@ import org.apache.hadoop.util.GenericOptionsParser;
|
|
|
import org.apache.hadoop.util.JvmPauseMonitor;
|
|
|
import org.apache.hadoop.util.ServicePlugin;
|
|
|
import org.apache.hadoop.util.StringUtils;
|
|
|
+import org.apache.log4j.Appender;
|
|
|
+import org.apache.log4j.AsyncAppender;
|
|
|
import org.slf4j.Logger;
|
|
|
import org.slf4j.LoggerFactory;
|
|
|
|
|
|
+import javax.management.Attribute;
|
|
|
+import javax.management.AttributeList;
|
|
|
+import javax.management.MBeanAttributeInfo;
|
|
|
+import javax.management.MBeanInfo;
|
|
|
+import javax.management.MBeanServer;
|
|
|
+import javax.management.MalformedObjectNameException;
|
|
|
import javax.management.ObjectName;
|
|
|
|
|
|
import java.io.IOException;
|
|
|
import java.io.PrintStream;
|
|
|
+import java.lang.management.ManagementFactory;
|
|
|
import java.net.InetSocketAddress;
|
|
|
import java.net.URI;
|
|
|
import java.security.PrivilegedExceptionAction;
|
|
|
import java.util.ArrayList;
|
|
|
import java.util.Arrays;
|
|
|
import java.util.Collection;
|
|
|
+import java.util.Collections;
|
|
|
+import java.util.HashSet;
|
|
|
import java.util.List;
|
|
|
+import java.util.Set;
|
|
|
+import java.util.concurrent.ScheduledThreadPoolExecutor;
|
|
|
+import java.util.concurrent.TimeUnit;
|
|
|
import java.util.concurrent.atomic.AtomicBoolean;
|
|
|
|
|
|
import static org.apache.hadoop.fs.CommonConfigurationKeysPublic.FS_DEFAULT_NAME_KEY;
|
|
@@ -113,6 +130,8 @@ import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_HTTP_BIND_HOST_K
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_KERBEROS_INTERNAL_SPNEGO_PRINCIPAL_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_KERBEROS_PRINCIPAL_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_KEYTAB_FILE_KEY;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_METRICS_LOGGER_PERIOD_SECONDS_DEFAULT;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_METRICS_LOGGER_PERIOD_SECONDS_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_NAME_DIR_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_PLUGINS_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY;
|
|
@@ -303,7 +322,10 @@ public class NameNode implements NameNodeStatusMXBean {
|
|
|
LoggerFactory.getLogger("BlockStateChange");
|
|
|
public static final HAState ACTIVE_STATE = new ActiveState();
|
|
|
public static final HAState STANDBY_STATE = new StandbyState();
|
|
|
-
|
|
|
+
|
|
|
+ public static final Log MetricsLog =
|
|
|
+ LogFactory.getLog("NameNodeMetricsLog");
|
|
|
+
|
|
|
protected FSNamesystem namesystem;
|
|
|
protected final Configuration conf;
|
|
|
protected final NamenodeRole role;
|
|
@@ -329,6 +351,8 @@ public class NameNode implements NameNodeStatusMXBean {
|
|
|
private JvmPauseMonitor pauseMonitor;
|
|
|
private ObjectName nameNodeStatusBeanName;
|
|
|
SpanReceiverHost spanReceiverHost;
|
|
|
+ ScheduledThreadPoolExecutor metricsLoggerTimer;
|
|
|
+
|
|
|
/**
|
|
|
* The namenode address that clients will use to access this namenode
|
|
|
* or the name service. For HA configurations using logical URI, it
|
|
@@ -662,6 +686,68 @@ public class NameNode implements NameNodeStatusMXBean {
|
|
|
metrics.getJvmMetrics().setPauseMonitor(pauseMonitor);
|
|
|
|
|
|
startCommonServices(conf);
|
|
|
+ startMetricsLogger(conf);
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Start a timer to periodically write NameNode metrics to the log
|
|
|
+ * file. This behavior can be disabled by configuration.
|
|
|
+ * @param conf
|
|
|
+ */
|
|
|
+ protected void startMetricsLogger(Configuration conf) {
|
|
|
+ long metricsLoggerPeriodSec =
|
|
|
+ conf.getInt(DFS_NAMENODE_METRICS_LOGGER_PERIOD_SECONDS_KEY,
|
|
|
+ DFS_NAMENODE_METRICS_LOGGER_PERIOD_SECONDS_DEFAULT);
|
|
|
+
|
|
|
+ if (metricsLoggerPeriodSec <= 0) {
|
|
|
+ return;
|
|
|
+ }
|
|
|
+
|
|
|
+ makeMetricsLoggerAsync();
|
|
|
+
|
|
|
+ // Schedule the periodic logging.
|
|
|
+ metricsLoggerTimer = new ScheduledThreadPoolExecutor(1);
|
|
|
+ metricsLoggerTimer.setExecuteExistingDelayedTasksAfterShutdownPolicy(
|
|
|
+ false);
|
|
|
+ metricsLoggerTimer.scheduleWithFixedDelay(new MetricsLoggerTask(),
|
|
|
+ metricsLoggerPeriodSec,
|
|
|
+ metricsLoggerPeriodSec,
|
|
|
+ TimeUnit.SECONDS);
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Make the metrics logger async and add all pre-existing appenders
|
|
|
+ * to the async appender.
|
|
|
+ */
|
|
|
+ private static void makeMetricsLoggerAsync() {
|
|
|
+ if (!(MetricsLog instanceof Log4JLogger)) {
|
|
|
+ LOG.warn(
|
|
|
+ "Metrics logging will not be async since the logger is not log4j");
|
|
|
+ return;
|
|
|
+ }
|
|
|
+ org.apache.log4j.Logger logger = ((Log4JLogger) MetricsLog).getLogger();
|
|
|
+ logger.setAdditivity(false); // Don't pollute NN logs with metrics dump
|
|
|
+
|
|
|
+ @SuppressWarnings("unchecked")
|
|
|
+ List<Appender> appenders = Collections.list(logger.getAllAppenders());
|
|
|
+ // failsafe against trying to async it more than once
|
|
|
+ if (!appenders.isEmpty() && !(appenders.get(0) instanceof AsyncAppender)) {
|
|
|
+ AsyncAppender asyncAppender = new AsyncAppender();
|
|
|
+ // change logger to have an async appender containing all the
|
|
|
+ // previously configured appenders
|
|
|
+ for (Appender appender : appenders) {
|
|
|
+ logger.removeAppender(appender);
|
|
|
+ asyncAppender.addAppender(appender);
|
|
|
+ }
|
|
|
+ logger.addAppender(asyncAppender);
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ protected void stopMetricsLogger() {
|
|
|
+ if (metricsLoggerTimer != null) {
|
|
|
+ metricsLoggerTimer.shutdown();
|
|
|
+ metricsLoggerTimer = null;
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
/**
|
|
@@ -865,6 +951,7 @@ public class NameNode implements NameNodeStatusMXBean {
|
|
|
} catch (ServiceFailedException e) {
|
|
|
LOG.warn("Encountered exception while exiting state ", e);
|
|
|
} finally {
|
|
|
+ stopMetricsLogger();
|
|
|
stopCommonServices();
|
|
|
if (metrics != null) {
|
|
|
metrics.shutdown();
|
|
@@ -1830,4 +1917,91 @@ public class NameNode implements NameNodeStatusMXBean {
|
|
|
break;
|
|
|
}
|
|
|
}
|
|
|
+
|
|
|
+ private static class MetricsLoggerTask implements Runnable {
|
|
|
+ private static final int MAX_LOGGED_VALUE_LEN = 128;
|
|
|
+ private static ObjectName OBJECT_NAME = null;
|
|
|
+
|
|
|
+ static {
|
|
|
+ try {
|
|
|
+ OBJECT_NAME = new ObjectName("Hadoop:*");
|
|
|
+ } catch (MalformedObjectNameException m) {
|
|
|
+ // This should not occur in practice since we pass
|
|
|
+ // a valid pattern to the constructor above.
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Write NameNode metrics to the metrics appender when invoked.
|
|
|
+ */
|
|
|
+ @Override
|
|
|
+ public void run() {
|
|
|
+ // Skip querying metrics if there are no known appenders.
|
|
|
+ if (!MetricsLog.isInfoEnabled() ||
|
|
|
+ !hasAppenders(MetricsLog) ||
|
|
|
+ OBJECT_NAME == null) {
|
|
|
+ return;
|
|
|
+ }
|
|
|
+
|
|
|
+ MetricsLog.info(" >> Begin NameNode metrics dump");
|
|
|
+ final MBeanServer server = ManagementFactory.getPlatformMBeanServer();
|
|
|
+
|
|
|
+ // Iterate over each MBean.
|
|
|
+ for (final ObjectName mbeanName : server.queryNames(OBJECT_NAME, null)) {
|
|
|
+ try {
|
|
|
+ MBeanInfo mBeanInfo = server.getMBeanInfo(mbeanName);
|
|
|
+ final String mBeanNameName = MBeans.getMbeanNameName(mbeanName);
|
|
|
+ final Set<String> attributeNames = getFilteredAttributes(mBeanInfo);
|
|
|
+
|
|
|
+ final AttributeList attributes =
|
|
|
+ server.getAttributes(mbeanName,
|
|
|
+ attributeNames.toArray(new String[attributeNames.size()]));
|
|
|
+
|
|
|
+ for (Object o : attributes) {
|
|
|
+ final Attribute attribute = (Attribute) o;
|
|
|
+ final Object value = attribute.getValue();
|
|
|
+ final String valueStr =
|
|
|
+ (value != null) ? value.toString() : "null";
|
|
|
+ // Truncate the value if it is too long
|
|
|
+ MetricsLog.info(mBeanNameName + ":" + attribute.getName() + "=" +
|
|
|
+ (valueStr.length() < MAX_LOGGED_VALUE_LEN ? valueStr :
|
|
|
+ valueStr.substring(0, MAX_LOGGED_VALUE_LEN) + "..."));
|
|
|
+ }
|
|
|
+ } catch (Exception e) {
|
|
|
+ MetricsLog.error("Failed to get NameNode metrics for mbean " +
|
|
|
+ mbeanName.toString(), e);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ MetricsLog.info(" << End NameNode metrics dump");
|
|
|
+ }
|
|
|
+
|
|
|
+ private static boolean hasAppenders(Log logger) {
|
|
|
+ if (!(logger instanceof Log4JLogger)) {
|
|
|
+ // Don't bother trying to determine the presence of appenders.
|
|
|
+ return true;
|
|
|
+ }
|
|
|
+ Log4JLogger log4JLogger = ((Log4JLogger) MetricsLog);
|
|
|
+ return log4JLogger.getLogger().getAllAppenders().hasMoreElements();
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Get the list of attributes for the MBean, filtering out a few
|
|
|
+ * attribute types.
|
|
|
+ */
|
|
|
+ private static Set<String> getFilteredAttributes(
|
|
|
+ MBeanInfo mBeanInfo) {
|
|
|
+ Set<String> attributeNames = new HashSet<>();
|
|
|
+ for (MBeanAttributeInfo attributeInfo : mBeanInfo.getAttributes()) {
|
|
|
+ if (!attributeInfo.getType().equals(
|
|
|
+ "javax.management.openmbean.TabularData") &&
|
|
|
+ !attributeInfo.getType().equals(
|
|
|
+ "javax.management.openmbean.CompositeData") &&
|
|
|
+ !attributeInfo.getType().equals(
|
|
|
+ "[Ljavax.management.openmbean.CompositeData;")) {
|
|
|
+ attributeNames.add(attributeInfo.getName());
|
|
|
+ }
|
|
|
+ }
|
|
|
+ return attributeNames;
|
|
|
+ }
|
|
|
+ }
|
|
|
}
|