You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by ay...@apache.org on 2019/03/23 06:59:07 UTC

[hadoop] branch HDFS-13891 updated: HDFS-14388. RBF: Prevent loading metric system when disabled. Contributed by Inigo Goiri.

This is an automated email from the ASF dual-hosted git repository.

ayushsaxena pushed a commit to branch HDFS-13891
in repository https://gitbox.apache.org/repos/asf/hadoop.git


The following commit(s) were added to refs/heads/HDFS-13891 by this push:
     new 42dc40e  HDFS-14388. RBF: Prevent loading metric system when disabled. Contributed by Inigo Goiri.
42dc40e is described below

commit 42dc40eb474241b05d07bae967b177f1abf1d845
Author: Ayush Saxena <ay...@apache.org>
AuthorDate: Sat Mar 23 12:16:31 2019 +0530

    HDFS-14388. RBF: Prevent loading metric system when disabled. Contributed by Inigo Goiri.
---
 .../metrics/FederationRPCPerformanceMonitor.java   | 36 ++++++++++----
 .../federation/metrics/NullStateStoreMetrics.java  | 56 ++++++++++++++++++++++
 .../federation/metrics/StateStoreMetrics.java      |  4 +-
 .../server/federation/router/RouterRpcServer.java  | 19 +++++---
 .../server/federation/store/StateStoreService.java | 33 ++++++++-----
 5 files changed, 118 insertions(+), 30 deletions(-)

diff --git a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/FederationRPCPerformanceMonitor.java b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/FederationRPCPerformanceMonitor.java
index bae83aa..5f06f59 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/FederationRPCPerformanceMonitor.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/FederationRPCPerformanceMonitor.java
@@ -154,47 +154,65 @@ public class FederationRPCPerformanceMonitor implements RouterRpcMonitor {
 
   @Override
   public void proxyOpFailureCommunicate() {
-    metrics.incrProxyOpFailureCommunicate();
+    if (metrics != null) {
+      metrics.incrProxyOpFailureCommunicate();
+    }
   }
 
   @Override
   public void proxyOpFailureClientOverloaded() {
-    metrics.incrProxyOpFailureClientOverloaded();
+    if (metrics != null) {
+      metrics.incrProxyOpFailureClientOverloaded();
+    }
   }
 
   @Override
   public void proxyOpNotImplemented() {
-    metrics.incrProxyOpNotImplemented();
+    if (metrics != null) {
+      metrics.incrProxyOpNotImplemented();
+    }
   }
 
   @Override
   public void proxyOpRetries() {
-    metrics.incrProxyOpRetries();
+    if (metrics != null) {
+      metrics.incrProxyOpRetries();
+    }
   }
 
   @Override
   public void proxyOpNoNamenodes() {
-    metrics.incrProxyOpNoNamenodes();
+    if (metrics != null) {
+      metrics.incrProxyOpNoNamenodes();
+    }
   }
 
   @Override
   public void routerFailureStateStore() {
-    metrics.incrRouterFailureStateStore();
+    if (metrics != null) {
+      metrics.incrRouterFailureStateStore();
+    }
   }
 
   @Override
   public void routerFailureSafemode() {
-    metrics.incrRouterFailureSafemode();
+    if (metrics != null) {
+      metrics.incrRouterFailureSafemode();
+    }
   }
 
   @Override
   public void routerFailureReadOnly() {
-    metrics.incrRouterFailureReadOnly();
+    if (metrics != null) {
+      metrics.incrRouterFailureReadOnly();
+    }
   }
 
   @Override
   public void routerFailureLocked() {
-    metrics.incrRouterFailureLocked();
+    if (metrics != null) {
+      metrics.incrRouterFailureLocked();
+    }
   }
 
 
diff --git a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/NullStateStoreMetrics.java b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/NullStateStoreMetrics.java
new file mode 100644
index 0000000..d74aed9
--- /dev/null
+++ b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/NullStateStoreMetrics.java
@@ -0,0 +1,56 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hdfs.server.federation.metrics;
+
+/**
+ * Implementation of the State Store metrics which does not do anything.
+ * This is used when the metrics are disabled (e.g., tests).
+ */
+public class NullStateStoreMetrics extends StateStoreMetrics {
+  public void addRead(long latency) {}
+  public long getReadOps() {
+    return -1;
+  }
+  public double getReadAvg() {
+    return -1;
+  }
+  public void addWrite(long latency) {}
+  public long getWriteOps() {
+    return -1;
+  }
+  public double getWriteAvg() {
+    return -1;
+  }
+  public void addFailure(long latency) {  }
+  public long getFailureOps() {
+    return -1;
+  }
+  public double getFailureAvg() {
+    return -1;
+  }
+  public void addRemove(long latency) {}
+  public long getRemoveOps() {
+    return -1;
+  }
+  public double getRemoveAvg() {
+    return -1;
+  }
+  public void setCacheSize(String name, int size) {}
+  public void reset() {}
+  public void shutdown() {}
+}
diff --git a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/StateStoreMetrics.java b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/StateStoreMetrics.java
index 09253a2..64bb108 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/StateStoreMetrics.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/StateStoreMetrics.java
@@ -39,7 +39,7 @@ import com.google.common.annotations.VisibleForTesting;
  */
 @Metrics(name = "StateStoreActivity", about = "Router metrics",
     context = "dfs")
-public final class StateStoreMetrics implements StateStoreMBean {
+public class StateStoreMetrics implements StateStoreMBean {
 
   private final MetricsRegistry registry = new MetricsRegistry("router");
 
@@ -54,6 +54,8 @@ public final class StateStoreMetrics implements StateStoreMBean {
 
   private Map<String, MutableGaugeInt> cacheSizes;
 
+  protected StateStoreMetrics() {}
+
   private StateStoreMetrics(Configuration conf) {
     registry.tag(SessionId, "RouterSession");
     registry.tag(ProcessName, "Router");
diff --git a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/RouterRpcServer.java b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/RouterRpcServer.java
index e4ea58b..739a2ff 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/RouterRpcServer.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/RouterRpcServer.java
@@ -304,12 +304,17 @@ public class RouterRpcServer extends AbstractService
     this.rpcAddress = new InetSocketAddress(
         confRpcAddress.getHostName(), listenAddress.getPort());
 
-    // Create metrics monitor
-    Class<? extends RouterRpcMonitor> rpcMonitorClass = this.conf.getClass(
-        RBFConfigKeys.DFS_ROUTER_METRICS_CLASS,
-        RBFConfigKeys.DFS_ROUTER_METRICS_CLASS_DEFAULT,
-        RouterRpcMonitor.class);
-    this.rpcMonitor = ReflectionUtils.newInstance(rpcMonitorClass, conf);
+    if (conf.getBoolean(RBFConfigKeys.DFS_ROUTER_METRICS_ENABLE,
+        RBFConfigKeys.DFS_ROUTER_METRICS_ENABLE_DEFAULT)) {
+      // Create metrics monitor
+      Class<? extends RouterRpcMonitor> rpcMonitorClass = this.conf.getClass(
+          RBFConfigKeys.DFS_ROUTER_METRICS_CLASS,
+          RBFConfigKeys.DFS_ROUTER_METRICS_CLASS_DEFAULT,
+          RouterRpcMonitor.class);
+      this.rpcMonitor = ReflectionUtils.newInstance(rpcMonitorClass, conf);
+    } else {
+      this.rpcMonitor = null;
+    }
 
     // Create the client
     this.rpcClient = new RouterRpcClient(this.conf, this.router,
@@ -326,7 +331,7 @@ public class RouterRpcServer extends AbstractService
     this.conf = configuration;
 
     if (this.rpcMonitor == null) {
-      LOG.error("Cannot instantiate Router RPC metrics class");
+      LOG.info("Do not start Router RPC metrics");
     } else {
       this.rpcMonitor.init(this.conf, this, this.router.getStateStore());
     }
diff --git a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/store/StateStoreService.java b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/store/StateStoreService.java
index c55f4cd..37b62fb 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/store/StateStoreService.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/store/StateStoreService.java
@@ -33,6 +33,7 @@ import javax.management.StandardMBean;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hdfs.server.federation.metrics.NullStateStoreMetrics;
 import org.apache.hadoop.hdfs.server.federation.metrics.StateStoreMBean;
 import org.apache.hadoop.hdfs.server.federation.metrics.StateStoreMetrics;
 import org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys;
@@ -172,19 +173,25 @@ public class StateStoreService extends CompositeService {
     this.cacheUpdater = new StateStoreCacheUpdateService(this);
     addService(this.cacheUpdater);
 
-    // Create metrics for the State Store
-    this.metrics = StateStoreMetrics.create(conf);
-
-    // Adding JMX interface
-    try {
-      StandardMBean bean = new StandardMBean(metrics, StateStoreMBean.class);
-      ObjectName registeredObject =
-          MBeans.register("Router", "StateStore", bean);
-      LOG.info("Registered StateStoreMBean: {}", registeredObject);
-    } catch (NotCompliantMBeanException e) {
-      throw new RuntimeException("Bad StateStoreMBean setup", e);
-    } catch (MetricsException e) {
-      LOG.error("Failed to register State Store bean {}", e.getMessage());
+    if (conf.getBoolean(RBFConfigKeys.DFS_ROUTER_METRICS_ENABLE,
+        RBFConfigKeys.DFS_ROUTER_METRICS_ENABLE_DEFAULT)) {
+      // Create metrics for the State Store
+      this.metrics = StateStoreMetrics.create(conf);
+
+      // Adding JMX interface
+      try {
+        StandardMBean bean = new StandardMBean(metrics, StateStoreMBean.class);
+        ObjectName registeredObject =
+            MBeans.register("Router", "StateStore", bean);
+        LOG.info("Registered StateStoreMBean: {}", registeredObject);
+      } catch (NotCompliantMBeanException e) {
+        throw new RuntimeException("Bad StateStoreMBean setup", e);
+      } catch (MetricsException e) {
+        LOG.error("Failed to register State Store bean {}", e.getMessage());
+      }
+    } else {
+      LOG.info("State Store metrics not enabled");
+      this.metrics = new NullStateStoreMetrics();
     }
 
     super.serviceInit(this.conf);


---------------------------------------------------------------------
To unsubscribe, e-mail: common-commits-unsubscribe@hadoop.apache.org
For additional commands, e-mail: common-commits-help@hadoop.apache.org