Procházet zdrojové kódy

HDFS-16882. RBF: Add cache hit rate metric in MountTableResolver#getDestinationForPath (#5276) (#5423). Contributed by farmmamba

Reviewed-by: Inigo Goiri <inigoiri@apache.org>
Reviewed-by: Tao Li <tomscut@apache.org>
Signed-off-by: Ayush Saxena <ayushsaxena@apache.org>
hfutatzhanghb před 1 rokem
rodič
revize
c2385c021b

+ 14 - 0
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/metrics/StateStoreMetrics.java

@@ -30,6 +30,7 @@ import org.apache.hadoop.metrics2.annotation.Metrics;
 import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
 import org.apache.hadoop.metrics2.lib.MetricsRegistry;
 import org.apache.hadoop.metrics2.lib.MutableGaugeInt;
+import org.apache.hadoop.metrics2.lib.MutableGaugeLong;
 import org.apache.hadoop.metrics2.lib.MutableRate;
 
 import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
@@ -136,6 +137,19 @@ public class StateStoreMetrics implements StateStoreMBean {
     counter.set(size);
   }
 
+  /**
+   * set the count of the location cache access information.
+   * @param name Name of the record.
+   * @param count count of the record.
+   */
+  public void setLocationCache(String name, long count) {
+    MutableGaugeLong counter = (MutableGaugeLong) registry.get(name);
+    if (counter == null) {
+      counter = registry.newGauge(name, name, count);
+    }
+    counter.set(count);
+  }
+
   @VisibleForTesting
   public void reset() {
     reads.resetMinMax();

+ 20 - 2
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/resolver/MountTableResolver.java

@@ -42,6 +42,7 @@ import java.util.concurrent.Callable;
 import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ConcurrentMap;
 import java.util.concurrent.ExecutionException;
+import java.util.concurrent.atomic.LongAdder;
 import java.util.concurrent.locks.Lock;
 import java.util.concurrent.locks.ReadWriteLock;
 import java.util.concurrent.locks.ReentrantReadWriteLock;
@@ -51,6 +52,7 @@ import java.util.ArrayList;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.FileSystem;
+import org.apache.hadoop.hdfs.server.federation.metrics.StateStoreMetrics;
 import org.apache.hadoop.hdfs.server.federation.resolver.order.DestinationOrder;
 import org.apache.hadoop.hdfs.server.federation.router.Router;
 import org.apache.hadoop.hdfs.server.federation.router.RouterRpcServer;
@@ -97,6 +99,8 @@ public class MountTableResolver
   private final TreeMap<String, MountTable> tree = new TreeMap<>();
   /** Path -> Remote location. */
   private final Cache<String, PathLocation> locationCache;
+  private final LongAdder locCacheMiss = new LongAdder();
+  private final LongAdder locCacheAccess = new LongAdder();
 
   /** Default nameservice when no mount matches the math. */
   private String defaultNameService = "";
@@ -408,6 +412,9 @@ public class MountTableResolver
           mountTable.getMountTableEntries(request);
       List<MountTable> records = response.getEntries();
       refreshEntries(records);
+      StateStoreMetrics metrics = this.getMountTableStore().getDriver().getMetrics();
+      metrics.setLocationCache("locationCacheMissed", this.getLocCacheMiss().sum());
+      metrics.setLocationCache("locationCacheAccessed", this.getLocCacheAccess().sum());
     } catch (IOException e) {
       LOG.error("Cannot fetch mount table entries from State Store", e);
       return false;
@@ -441,9 +448,12 @@ public class MountTableResolver
       if (this.locationCache == null) {
         res = lookupLocation(processTrashPath(path));
       } else {
-        Callable<? extends PathLocation> meh = (Callable<PathLocation>) () ->
-            lookupLocation(processTrashPath(path));
+        Callable<? extends PathLocation> meh = (Callable<PathLocation>) () -> {
+          this.getLocCacheMiss().increment();
+          return lookupLocation(processTrashPath(path));
+        };
         res = this.locationCache.get(processTrashPath(path), meh);
+        this.getLocCacheAccess().increment();
       }
       if (isTrashPath(path)) {
         List<RemoteLocation> remoteLocations = new ArrayList<>();
@@ -734,4 +744,12 @@ public class MountTableResolver
   public void setDisabled(boolean disable) {
     this.disabled = disable;
   }
+
+  public LongAdder getLocCacheMiss() {
+    return locCacheMiss;
+  }
+
+  public LongAdder getLocCacheAccess() {
+    return locCacheAccess;
+  }
 }

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/store/RecordStore.java

@@ -73,7 +73,7 @@ public abstract class RecordStore<R extends BaseRecord> {
    *
    * @return State Store driver.
    */
-  protected StateStoreDriver getDriver() {
+  public StateStoreDriver getDriver() {
     return this.driver;
   }
 

+ 38 - 1
hadoop-hdfs-project/hadoop-hdfs-rbf/src/test/java/org/apache/hadoop/hdfs/server/federation/resolver/TestMountTableResolver.java

@@ -729,4 +729,41 @@ public class TestMountTableResolver {
     assertEquals("2->/testInvalidateCache/foo", mountTable
         .getDestinationForPath("/testInvalidateCache/foo").toString());
   }
-}
+
+  /**
+   * Test location cache hit when get destination for path.
+   */
+  @Test
+  public void testLocationCacheHitrate() throws Exception {
+    List<MountTable> entries = new ArrayList<>();
+
+    // Add entry and test location cache
+    Map<String, String> map1 = getMountTableEntry("1", "/testlocationcache");
+    MountTable entry1 = MountTable.newInstance("/testlocationcache", map1);
+    entries.add(entry1);
+
+    Map<String, String> map2 = getMountTableEntry("2",
+        "/anothertestlocationcache");
+    MountTable entry2 = MountTable.newInstance("/anothertestlocationcache",
+        map2);
+    entries.add(entry2);
+
+    mountTable.refreshEntries(entries);
+    mountTable.getLocCacheAccess().reset();
+    mountTable.getLocCacheMiss().reset();
+    assertEquals("1->/testlocationcache",
+        mountTable.getDestinationForPath("/testlocationcache").toString());
+    assertEquals("2->/anothertestlocationcache",
+        mountTable.getDestinationForPath("/anothertestlocationcache")
+            .toString());
+
+    assertEquals(2, mountTable.getLocCacheMiss().intValue());
+    assertEquals("1->/testlocationcache",
+        mountTable.getDestinationForPath("/testlocationcache").toString());
+    assertEquals(3, mountTable.getLocCacheAccess().intValue());
+
+    // Cleanup before exit
+    mountTable.removeEntry("/testlocationcache");
+    mountTable.removeEntry("/anothertestlocationcache");
+  }
+}