Prechádzať zdrojové kódy

HDFS-12807. Ozone: Expose RockDB stats via JMX for Ozone metadata stores. Contributed by Elek, Marton.

Anu Engineer 7 rokov pred
rodič
commit
8023ba50b5

+ 8 - 0
hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java

@@ -91,6 +91,14 @@ public final class OzoneConfigKeys {
   public static final String OZONE_METADATA_STORE_IMPL_DEFAULT =
       OZONE_METADATA_STORE_IMPL_ROCKSDB;
 
+  public static final String OZONE_METADATA_STORE_ROCKSDB_STATISTICS =
+      "ozone.metastore.rocksdb.statistics";
+
+  public static final String  OZONE_METADATA_STORE_ROCKSDB_STATISTICS_DEFAULT =
+      "ALL";
+  public static final String OZONE_METADATA_STORE_ROCKSDB_STATISTICS_OFF =
+      "OFF";
+
   public static final String OZONE_CONTAINER_CACHE_SIZE =
       "ozone.container.cache.size";
   public static final int OZONE_CONTAINER_CACHE_DEFAULT = 1024;

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs/pom.xml

@@ -190,7 +190,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
     <dependency>
       <groupId>org.rocksdb</groupId>
       <artifactId>rocksdbjni</artifactId>
-      <version>5.5.5</version>
+      <version>5.8.0</version>
     </dependency>
     <dependency>
       <groupId>io.swagger</groupId>

+ 21 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/utils/MetadataStoreBuilder.java

@@ -20,6 +20,12 @@ package org.apache.hadoop.utils;
 
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.ozone.OzoneConfigKeys;
+import static org.apache.hadoop.ozone.OzoneConfigKeys
+    .OZONE_METADATA_STORE_ROCKSDB_STATISTICS;
+import static org.apache.hadoop.ozone.OzoneConfigKeys
+    .OZONE_METADATA_STORE_ROCKSDB_STATISTICS_DEFAULT;
+import static org.apache.hadoop.ozone.OzoneConfigKeys
+    .OZONE_METADATA_STORE_ROCKSDB_STATISTICS_OFF;
 import org.iq80.leveldb.Options;
 import org.rocksdb.BlockBasedTableConfig;
 
@@ -29,6 +35,9 @@ import java.io.IOException;
 import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_STORE_IMPL_LEVELDB;
 import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_STORE_IMPL_ROCKSDB;
 
+import org.rocksdb.Statistics;
+import org.rocksdb.StatsLevel;
+
 /**
  * Builder for metadata store.
  */
@@ -91,6 +100,18 @@ public class MetadataStoreBuilder {
         tableConfig.setBlockCacheSize(cacheSize);
         opts.setTableFormatConfig(tableConfig);
       }
+
+      String rocksDbStat = conf == null ?
+          OZONE_METADATA_STORE_ROCKSDB_STATISTICS_DEFAULT :
+          conf.getTrimmed(OZONE_METADATA_STORE_ROCKSDB_STATISTICS,
+              OZONE_METADATA_STORE_ROCKSDB_STATISTICS_DEFAULT);
+
+      if (!rocksDbStat.equals(OZONE_METADATA_STORE_ROCKSDB_STATISTICS_OFF)) {
+        Statistics statistics = new Statistics();
+        statistics.setStatsLevel(StatsLevel.valueOf(rocksDbStat));
+        opts = opts.setStatistics(statistics);
+
+      }
       store = new RocksDBStore(dbFile, opts);
     } else {
       throw new IllegalArgumentException("Invalid argument for "

+ 30 - 3
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/utils/RocksDBStore.java

@@ -21,8 +21,13 @@ package org.apache.hadoop.utils;
 import com.google.common.base.Preconditions;
 import org.apache.commons.io.FileUtils;
 import org.apache.commons.lang3.tuple.ImmutablePair;
+import org.apache.hadoop.metrics2.util.MBeans;
+import org.apache.ratis.shaded.com.google.common.annotations.VisibleForTesting;
+import org.rocksdb.DBOptions;
 import org.rocksdb.RocksIterator;
 import org.rocksdb.Options;
+import org.rocksdb.Statistics;
+import org.rocksdb.StatsLevel;
 import org.rocksdb.WriteOptions;
 import org.rocksdb.RocksDB;
 import org.rocksdb.RocksDBException;
@@ -31,8 +36,10 @@ import org.rocksdb.DbPath;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import javax.management.ObjectName;
 import java.io.File;
 import java.io.IOException;
+import java.util.HashMap;
 import java.util.Map;
 import java.util.List;
 import java.util.ArrayList;
@@ -51,8 +58,10 @@ public class RocksDBStore implements MetadataStore {
   private File dbLocation;
   private WriteOptions writeOptions;
   private Options dbOptions;
+  private ObjectName statMBeanName;
 
-  public RocksDBStore(File dbFile, Options options) throws IOException {
+  public RocksDBStore(File dbFile, Options options)
+      throws IOException {
     Preconditions.checkNotNull(dbFile, "DB file location cannot be null");
     RocksDB.loadLibrary();
     dbOptions = options;
@@ -61,10 +70,18 @@ public class RocksDBStore implements MetadataStore {
     writeOptions.setSync(true);
     writeOptions.setNoSlowdown(true);
     try {
+
       db = RocksDB.open(dbOptions, dbLocation.getAbsolutePath());
+      if (dbOptions.statistics() != null) {
+
+        Map<String, String> jmxProperties = new HashMap<String, String>();
+        jmxProperties.put("dbName", dbFile.getName());
+        statMBeanName = MBeans.register("Ozone", "RocksDbStore", jmxProperties,
+            new RocksDBStoreMBean(dbOptions.statistics()));
+      }
     } catch (RocksDBException e) {
-      throw new IOException("Failed init RocksDB, db path : "
-          + dbFile.getAbsolutePath(), e);
+      throw new IOException(
+          "Failed init RocksDB, db path : " + dbFile.getAbsolutePath(), e);
     }
 
     if (LOG.isDebugEnabled()) {
@@ -349,8 +366,18 @@ public class RocksDBStore implements MetadataStore {
 
   @Override
   public void close() throws IOException {
+    if (statMBeanName != null) {
+      MBeans.unregister(statMBeanName);
+    }
     if (db != null) {
       db.close();
     }
+
+  }
+
+  @VisibleForTesting
+  protected ObjectName getStatMBeanName() {
+    return statMBeanName;
   }
+
 }

+ 148 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/utils/RocksDBStoreMBean.java

@@ -0,0 +1,148 @@
+/*
+ * 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.utils;
+
+import org.rocksdb.HistogramData;
+import org.rocksdb.HistogramType;
+import org.rocksdb.Statistics;
+import org.rocksdb.TickerType;
+
+import javax.management.Attribute;
+import javax.management.AttributeList;
+import javax.management.AttributeNotFoundException;
+import javax.management.DynamicMBean;
+import javax.management.InvalidAttributeValueException;
+import javax.management.MBeanAttributeInfo;
+import javax.management.MBeanException;
+import javax.management.MBeanInfo;
+import javax.management.ReflectionException;
+import java.lang.reflect.InvocationTargetException;
+import java.lang.reflect.Method;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Set;
+import java.util.stream.Stream;
+
+/**
+ * Adapter JMX bean to publish all the Rocksdb metrics.
+ */
+public class RocksDBStoreMBean implements DynamicMBean {
+
+  private Statistics statistics;
+
+  private Set<String> histogramAttributes = new HashSet<>();
+
+  public RocksDBStoreMBean(Statistics statistics) {
+    this.statistics = statistics;
+    histogramAttributes.add("Average");
+    histogramAttributes.add("Median");
+    histogramAttributes.add("Percentile95");
+    histogramAttributes.add("Percentile99");
+    histogramAttributes.add("StandardDeviation");
+  }
+
+  @Override
+  public Object getAttribute(String attribute)
+      throws AttributeNotFoundException, MBeanException, ReflectionException {
+    for (String histogramAttribute : histogramAttributes) {
+      if (attribute.endsWith("_" + histogramAttribute.toUpperCase())) {
+        String keyName = attribute
+            .substring(0, attribute.length() - histogramAttribute.length() -1);
+        try {
+          HistogramData histogram =
+              statistics.getHistogramData(HistogramType.valueOf(keyName));
+          try {
+            Method method =
+                HistogramData.class.getMethod("get" + histogramAttribute);
+            return method.invoke(histogram);
+          } catch (Exception e) {
+            throw new ReflectionException(e,
+                "Can't read attribute " + attribute);
+          }
+        } catch (IllegalArgumentException exception) {
+          throw new AttributeNotFoundException(
+              "No such attribute in RocksDB stats: " + attribute);
+        }
+      }
+    }
+    try {
+      return statistics.getTickerCount(TickerType.valueOf(attribute));
+    } catch (IllegalArgumentException ex) {
+      throw new AttributeNotFoundException(
+          "No such attribute in RocksDB stats: " + attribute);
+    }
+  }
+
+  @Override
+  public void setAttribute(Attribute attribute)
+      throws AttributeNotFoundException, InvalidAttributeValueException,
+      MBeanException, ReflectionException {
+
+  }
+
+  @Override
+  public AttributeList getAttributes(String[] attributes) {
+    AttributeList result = new AttributeList();
+    for (String attributeName : attributes) {
+      try {
+        Object value = getAttribute(attributeName);
+        result.add(value);
+      } catch (Exception e) {
+        //TODO
+      }
+    }
+    return result;
+  }
+
+  @Override
+  public AttributeList setAttributes(AttributeList attributes) {
+    return null;
+  }
+
+  @Override
+  public Object invoke(String actionName, Object[] params, String[] signature)
+      throws MBeanException, ReflectionException {
+    return null;
+  }
+
+  @Override
+  public MBeanInfo getMBeanInfo() {
+
+    List<MBeanAttributeInfo> attributes = new ArrayList<>();
+    for (TickerType tickerType : TickerType.values()) {
+      attributes.add(new MBeanAttributeInfo(tickerType.name(), "long",
+          "RocksDBStat: " + tickerType.name(), true, false, false));
+    }
+    for (HistogramType histogramType : HistogramType.values()) {
+      for (String histogramAttribute : histogramAttributes) {
+        attributes.add(new MBeanAttributeInfo(
+            histogramType.name() + "_" + histogramAttribute.toUpperCase(),
+            "long", "RocksDBStat: " + histogramType.name(), true, false,
+            false));
+      }
+    }
+
+    return new MBeanInfo("", "RocksDBStat",
+        attributes.toArray(new MBeanAttributeInfo[0]), null, null, null);
+
+  }
+}

+ 14 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/resources/ozone-default.xml

@@ -629,6 +629,20 @@
     </description>
   </property>
 
+  <property>
+    <name>ozone.metastore.rocksdb.statistics</name>
+    <value>ALL</value>
+    <tag>OZONE, KSM, SCM, STORAGE, PERFORMANCE</tag>
+    <description>
+      The statistics level of the rocksdb store. If you use any value from
+      org.rocksdb.StatsLevel (eg. ALL or EXCEPT_DETAILED_TIMERS), the rocksdb
+      statistics will be exposed over JMX bean with the choosed setting. Set
+      it to OFF to not initialize rocksdb statistics at all. Please note that
+      collection of statistics could have 5-10% performance penalty.
+      Check the rocksdb documentation for more details.
+    </description>
+  </property>
+
   <property>
     <name>ozone.scm.block.client.address</name>
     <value/>

+ 87 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/utils/TestRocksDBStoreMBean.java

@@ -0,0 +1,87 @@
+/**
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.utils;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.conf.OzoneConfiguration;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import static org.apache.hadoop.ozone.OzoneConfigKeys
+    .OZONE_METADATA_STORE_ROCKSDB_STATISTICS_OFF;
+import org.apache.hadoop.test.GenericTestUtils;
+import org.junit.Assert;
+import org.junit.Test;
+
+import javax.management.AttributeNotFoundException;
+import javax.management.MBeanServer;
+import java.io.File;
+import java.lang.management.ManagementFactory;
+
+public class TestRocksDBStoreMBean {
+
+  @Test
+  public void testJmxBeans() throws Exception {
+    File testDir =
+        GenericTestUtils.getTestDir(getClass().getSimpleName() + "-withstat");
+
+    Configuration conf = new OzoneConfiguration();
+    conf.set(OzoneConfigKeys.OZONE_METADATA_STORE_IMPL,
+        OzoneConfigKeys.OZONE_METADATA_STORE_IMPL_ROCKSDB);
+
+    RocksDBStore metadataStore =
+        (RocksDBStore) MetadataStoreBuilder.newBuilder().setConf(conf)
+            .setCreateIfMissing(true).setDbFile(testDir).build();
+
+    for (int i = 0; i < 10; i++) {
+      metadataStore.put("key".getBytes(), "value".getBytes());
+    }
+
+    MBeanServer platformMBeanServer =
+        ManagementFactory.getPlatformMBeanServer();
+    Thread.sleep(2000);
+
+    Object keysWritten = platformMBeanServer
+        .getAttribute(metadataStore.getStatMBeanName(), "NUMBER_KEYS_WRITTEN");
+
+    Assert.assertEquals(10l, keysWritten);
+
+    Object dbWriteAverage = platformMBeanServer
+        .getAttribute(metadataStore.getStatMBeanName(), "DB_WRITE_AVERAGE");
+    Assert.assertTrue((double) dbWriteAverage > 0);
+
+    metadataStore.close();
+
+  }
+
+  @Test()
+  public void testDisabledStat() throws Exception {
+    File testDir = GenericTestUtils
+        .getTestDir(getClass().getSimpleName() + "-withoutstat");
+
+    Configuration conf = new OzoneConfiguration();
+    conf.set(OzoneConfigKeys.OZONE_METADATA_STORE_IMPL,
+        OzoneConfigKeys.OZONE_METADATA_STORE_IMPL_ROCKSDB);
+    conf.set(OzoneConfigKeys.OZONE_METADATA_STORE_ROCKSDB_STATISTICS,
+        OzoneConfigKeys.OZONE_METADATA_STORE_ROCKSDB_STATISTICS_OFF);
+
+    RocksDBStore metadataStore =
+        (RocksDBStore) MetadataStoreBuilder.newBuilder().setConf(conf)
+            .setCreateIfMissing(true).setDbFile(testDir).build();
+
+    Assert.assertNull(metadataStore.getStatMBeanName());
+  }
+}