You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by ns...@apache.org on 2011/10/11 04:06:21 UTC

svn commit: r1181413 - in /hbase/branches/0.89: conf/ src/main/java/org/apache/hadoop/hbase/master/ src/main/java/org/apache/hadoop/hbase/master/metrics/ src/main/java/org/apache/hadoop/hbase/metrics/ src/main/java/org/apache/hadoop/hbase/regionserver/...

Author: nspiegelberg
Date: Tue Oct 11 02:06:21 2011
New Revision: 1181413

URL: http://svn.apache.org/viewvc?rev=1181413&view=rev
Log:
HBASE-3102: Add SplitLog Metrics + Refactor

Summary:
1. Modifications per peer review
2. Refactored to PersistentMetrics abstraction
3. Add SplitLog Time + Size to Master
4. Init min/max/avg on reset to previous avg.  This gets rid of the 'humps'
that we see on OpenTSDB

Test Plan:
mvn clean install -DskipTests
cluster testing

DiffCamp Revision: 172249
Reviewed By: kranganathan
Commenters: kannan, jgray
CC: jgray, nspiegelberg, achao, kannan, kranganathan
Tasks:
#418887: add flush metrics: number of flushes, flush times.

Revert Plan:
OK

Added:
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/metrics/PersistentMetricsTimeVaryingRate.java
Modified:
    hbase/branches/0.89/conf/hadoop-metrics.properties
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/ProcessServerShutdown.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/metrics/MasterMetrics.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStoreFlusher.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/metrics/RegionServerMetrics.java
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/wal/HLog.java

Modified: hbase/branches/0.89/conf/hadoop-metrics.properties
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/conf/hadoop-metrics.properties?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/conf/hadoop-metrics.properties (original)
+++ hbase/branches/0.89/conf/hadoop-metrics.properties Tue Oct 11 02:06:21 2011
@@ -16,6 +16,10 @@ hbase.class=org.apache.hadoop.metrics.sp
 # hbase.period=10
 # hbase.fileName=/tmp/metrics_hbase.log
 
+# HBase-specific configuration to reset long-running stats (e.g. compactions)
+# If this variable is left out, then the default is no expiration.
+hbase.extendedperiod = 3600
+
 # Configuration of the "hbase" context for ganglia
 # Pick one: Ganglia 3.0 (former) or Ganglia 3.1 (latter)
 # hbase.class=org.apache.hadoop.metrics.ganglia.GangliaContext

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/HMaster.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/HMaster.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/HMaster.java Tue Oct 11 02:06:21 2011
@@ -660,16 +660,21 @@ public class HMaster extends Thread impl
       if(this.serverManager.getServerInfo(serverName) == null) {
         LOG.info("Log folder doesn't belong " +
           "to a known region server, splitting");
+        long splitTime = 0, splitSize = 0;
+
         this.splitLogLock.lock();
         Path logDir =
           new Path(this.rootdir, HLog.getHLogDirectoryName(serverName));
         try {
           HLog.splitLog(this.rootdir, logDir, oldLogDir, this.fs, getConfiguration());
+          splitTime = HLog.lastSplitTime;
+          splitSize = HLog.lastSplitSize;
         } catch (IOException e) {
           LOG.error("Failed splitting " + logDir.toString(), e);
         } finally {
           this.splitLogLock.unlock();
         }
+        this.metrics.addSplit(splitTime, splitSize);
       } else {
         LOG.info("Log folder belongs to an existing region server");
       }

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/ProcessServerShutdown.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/ProcessServerShutdown.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/ProcessServerShutdown.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/ProcessServerShutdown.java Tue Oct 11 02:06:21 2011
@@ -32,6 +32,7 @@ import org.apache.hadoop.hbase.regionser
 import org.apache.hadoop.hbase.regionserver.wal.HLog;
 import org.apache.hadoop.hbase.util.Bytes;
 import org.apache.hadoop.hbase.master.RegionManager.RegionState;
+import org.apache.hadoop.hbase.master.metrics.MasterMetrics;
 
 import java.io.IOException;
 import java.util.ArrayList;
@@ -292,6 +293,8 @@ class ProcessServerShutdown extends Regi
     if (!logSplit) {
       // Process the old log file
       if (this.master.getFileSystem().exists(rsLogDir)) {
+        long splitTime = 0, splitSize = 0;
+
         if (!master.splitLogLock.tryLock()) {
           return false;
         }
@@ -299,9 +302,13 @@ class ProcessServerShutdown extends Regi
           HLog.splitLog(master.getRootDir(), rsLogDir,
               this.master.getOldLogDir(), this.master.getFileSystem(),
             this.master.getConfiguration());
+          splitTime = HLog.lastSplitTime;
+          splitSize = HLog.lastSplitSize;
         } finally {
           master.splitLogLock.unlock();
         }
+
+        this.master.getMetrics().addSplit(splitTime, splitSize);
       }
       logSplit = true;
     }

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/metrics/MasterMetrics.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/metrics/MasterMetrics.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/metrics/MasterMetrics.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/master/metrics/MasterMetrics.java Tue Oct 11 02:06:21 2011
@@ -17,14 +17,19 @@
  */
 package org.apache.hadoop.hbase.master.metrics;
 
+import java.io.IOException;
+
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.hbase.metrics.MetricsRate;
+import org.apache.hadoop.hbase.metrics.PersistentMetricsTimeVaryingRate;
+import org.apache.hadoop.metrics.ContextFactory;
 import org.apache.hadoop.metrics.MetricsContext;
 import org.apache.hadoop.metrics.MetricsRecord;
 import org.apache.hadoop.metrics.MetricsUtil;
 import org.apache.hadoop.metrics.Updater;
 import org.apache.hadoop.metrics.jvm.JvmMetrics;
+import org.apache.hadoop.metrics.util.MetricsLongValue;
 import org.apache.hadoop.metrics.util.MetricsRegistry;
 
 
@@ -40,12 +45,24 @@ public class MasterMetrics implements Up
   private final MetricsRecord metricsRecord;
   private final MetricsRegistry registry = new MetricsRegistry();
   private final MasterStatistics masterStatistics;
-  /*
+
+  private long lastUpdate = System.currentTimeMillis();
+  private long lastExtUpdate = System.currentTimeMillis();
+  private long extendedPeriod = 0;
+/*
    * Count of requests to the cluster since last call to metrics update
    */
   private final MetricsRate cluster_requests =
     new MetricsRate("cluster_requests", registry);
 
+  /** Time it takes to finish HLog.splitLog() */
+  final PersistentMetricsTimeVaryingRate splitTime =
+    new PersistentMetricsTimeVaryingRate("splitTime", registry);
+
+  /** Size of HLog files being split */
+  final PersistentMetricsTimeVaryingRate splitSize =
+    new PersistentMetricsTimeVaryingRate("splitSize", registry);
+
   public MasterMetrics(final String name) {
     MetricsContext context = MetricsUtil.getContext("hbase");
     metricsRecord = MetricsUtil.createRecord(context, "master");
@@ -56,6 +73,17 @@ public class MasterMetrics implements Up
     // expose the MBean for metrics
     masterStatistics = new MasterStatistics(this.registry);
 
+    // get custom attributes
+    try {
+      Object m =
+        ContextFactory.getFactory().getAttribute("hbase.extendedperiod");
+      if (m instanceof String) {
+        this.extendedPeriod = Long.parseLong((String) m)*1000;
+      }
+    } catch (IOException ioe) {
+      LOG.info("Couldn't load ContextFactory for Metrics config info");
+    }
+
     LOG.info("Initialized");
   }
 
@@ -71,7 +99,20 @@ public class MasterMetrics implements Up
    */
   public void doUpdates(MetricsContext unused) {
     synchronized (this) {
+      this.lastUpdate = System.currentTimeMillis();
+
+      // has the extended period for long-living stats elapsed?
+      if (this.extendedPeriod > 0 &&
+          this.lastUpdate - this.lastExtUpdate >= this.extendedPeriod) {
+        this.lastExtUpdate = this.lastUpdate;
+        this.splitTime.resetMinMaxAvg();
+        this.splitSize.resetMinMaxAvg();
+        this.resetAllMinMax();
+      }
+
       this.cluster_requests.pushMetric(metricsRecord);
+      this.splitTime.pushMetric(metricsRecord);
+      this.splitSize.pushMetric(metricsRecord);
     }
     this.metricsRecord.update();
   }
@@ -81,6 +122,16 @@ public class MasterMetrics implements Up
   }
 
   /**
+   * Record a single instance of a split
+   * @param time time that the split took
+   * @param size length of original HLogs that were split
+   */
+  public synchronized void addSplit(long time, long size) {
+    splitTime.inc(time);
+    splitSize.inc(size);
+  }
+
+  /**
    * @return Count of requests.
    */
   public float getRequests() {

Added: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/metrics/PersistentMetricsTimeVaryingRate.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/metrics/PersistentMetricsTimeVaryingRate.java?rev=1181413&view=auto
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/metrics/PersistentMetricsTimeVaryingRate.java (added)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/metrics/PersistentMetricsTimeVaryingRate.java Tue Oct 11 02:06:21 2011
@@ -0,0 +1,137 @@
+/**
+ * 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.hbase.metrics;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.metrics.MetricsRecord;
+import org.apache.hadoop.metrics.util.MetricsRegistry;
+import org.apache.hadoop.metrics.util.MetricsTimeVaryingRate;
+import org.apache.hadoop.util.StringUtils;
+
+/**
+ * This class extends MetricsTimeVaryingRate to let the metrics
+ * persist past a pushMetric() call
+ */
+public class PersistentMetricsTimeVaryingRate extends MetricsTimeVaryingRate {
+  protected static final Log LOG =
+    LogFactory.getLog("org.apache.hadoop.hbase.metrics");
+
+  protected boolean reset = false;
+  protected long lastOper = 0;
+  protected long totalOps = 0;
+
+  /**
+   * Constructor - create a new metric
+   * @param nam the name of the metrics to be used to publish the metric
+   * @param registry - where the metrics object will be registered
+   * @param description metrics description
+   */
+  public PersistentMetricsTimeVaryingRate(final String nam,
+      final MetricsRegistry registry,
+      final String description) {
+    super(nam, registry, description);
+  }
+
+  /**
+   * Constructor - create a new metric
+   * @param nam the name of the metrics to be used to publish the metric
+   * @param registry - where the metrics object will be registered
+   */
+  public PersistentMetricsTimeVaryingRate(final String nam,
+      MetricsRegistry registry) {
+    this(nam, registry, NO_DESCRIPTION);
+  }
+
+  /**
+   * Push updated metrics to the mr.
+   *
+   * Note this does NOT push to JMX
+   * (JMX gets the info via {@link #getPreviousIntervalAverageTime()} and
+   * {@link #getPreviousIntervalNumOps()}
+   *
+   * @param mr owner of this metric
+   */
+  @Override
+  public synchronized void pushMetric(final MetricsRecord mr) {
+    // this will reset the currentInterval & num_ops += prevInterval()
+    super.pushMetric(mr);
+    // since we're retaining prevInterval(), we don't want to do the incr
+    // instead, we want to set that value because we have absolute ops
+    try {
+      mr.setMetric(getName() + "_num_ops", totalOps);
+    } catch (Exception e) {
+      LOG.info("pushMetric failed for " + getName() + "\n" +
+          StringUtils.stringifyException(e));
+    }
+    if (reset) {
+      // use the previous avg as our starting min/max/avg
+      super.inc(getPreviousIntervalAverageTime());
+    } else {
+      // maintain the stats that pushMetric() cleared
+      maintainStats();
+    }
+  }
+
+  /**
+   * Increment the metrics for numOps operations
+   * @param numOps - number of operations
+   * @param time - time for numOps operations
+   */
+  @Override
+  public synchronized void inc(final int numOps, final long time) {
+    super.inc(numOps, time);
+    totalOps += numOps;
+  }
+
+  /**
+   * Increment the metrics for numOps operations
+   * @param time - time for numOps operations
+   */
+  @Override
+  public synchronized void inc(final long time) {
+    super.inc(time);
+    ++totalOps;
+  }
+
+  /**
+   * Rollover to a new interval
+   * NOTE: does not reset numOps.  this is an absolute value
+   */
+  public synchronized void resetMinMaxAvg() {
+    reset = true;
+  }
+
+  /* MetricsTimeVaryingRate will reset every time pushMetric() is called
+   * This is annoying for long-running stats that might not get a single
+   * operation in the polling period.  This function ensures that values
+   * for those stat entries don't get reset.
+   */
+  protected void maintainStats() {
+    int curOps = this.getPreviousIntervalNumOps();
+    if (curOps > 0) {
+      long curTime = this.getPreviousIntervalAverageTime();
+      long totalTime = curTime * curOps;
+      if (totalTime / curTime == curOps) {
+        super.inc(curOps, totalTime);
+      } else {
+        LOG.info("Stats for " + this.getName() + " overflowed! resetting");
+      }
+    }
+  }
+}

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java Tue Oct 11 02:06:21 2011
@@ -101,9 +101,8 @@ class CompactSplitThread extends Thread 
             if(!this.server.isStopRequested()) {
               // Don't interrupt us while we are working
               byte [] midKey = r.compactStores();
-              long lastCompactPeriod = r.getLastCompactPeriod();
-              if (lastCompactPeriod > 0) {  // 0 == compaction aborted
-                this.server.getMetrics().addCompaction(lastCompactPeriod);
+              if (r.getLastCompactInfo() != null) {  // compaction aborted?
+                this.server.getMetrics().addCompaction(r.getLastCompactInfo());
               }
               LOG.debug("Just finished a compaction. " +
                         " Current Compaction Queue Size: " +

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java Tue Oct 11 02:06:21 2011
@@ -177,7 +177,7 @@ public class HRegion implements HeapSize
    * major compaction.  Cleared each time through compaction code.
    */
   private volatile boolean forceMajorCompaction = false;
-  private long lastCompactPeriod = 0;
+  private Pair<Long,Long> lastCompactInfo = null;
 
   /*
    * Data structure of write state flags used coordinating flushes,
@@ -218,6 +218,8 @@ public class HRegion implements HeapSize
 
   final long memstoreFlushSize;
   private volatile long lastFlushTime;
+  private List<Pair<Long,Long>> recentFlushes
+    = new ArrayList<Pair<Long,Long>>();
   final FlushRequester flushListener;
   private final long blockingMemStoreSize;
   final long threadWakeFrequency;
@@ -607,9 +609,9 @@ public class HRegion implements HeapSize
     return this.fs;
   }
 
-  /** @return how long the last compaction took */
-  public long getLastCompactPeriod() {
-    return this.lastCompactPeriod;
+  /** @return how info about the last compaction <time, size> */
+  public Pair<Long, Long> getLastCompactInfo() {
+    return this.lastCompactInfo;
   }
 
   /** @return the last time the region was flushed */
@@ -617,6 +619,18 @@ public class HRegion implements HeapSize
     return this.lastFlushTime;
   }
 
+  /** @return how info about the last flushes <time, size> */
+  public List<Pair<Long,Long>> getRecentFlushInfo() {
+    // only MemStoreFlusher thread should be calling this, so read lock is okay
+    this.splitsAndClosesLock.readLock().lock();
+
+    List<Pair<Long,Long>> ret = this.recentFlushes;
+    this.recentFlushes = new ArrayList<Pair<Long,Long>>();
+
+    this.splitsAndClosesLock.readLock().unlock();
+    return ret;
+  }
+
   //////////////////////////////////////////////////////////////////////////////
   // HRegion maintenance.
   //
@@ -815,6 +829,7 @@ public class HRegion implements HeapSize
       return null;
     }
     splitsAndClosesLock.readLock().lock();
+    this.lastCompactInfo = null;
     try {
       byte [] splitRow = null;
       if (this.closed.get()) {
@@ -835,11 +850,13 @@ public class HRegion implements HeapSize
             "compaction on region " + this);
         long startTime = EnvironmentEdgeManager.currentTimeMillis();
         doRegionCompactionPrep();
+        long lastCompactSize = 0;
         long maxSize = -1;
         boolean completed = false;
         try {
           for (Store store: stores.values()) {
             final Store.StoreSize ss = store.compact(majorCompaction);
+            lastCompactSize += store.getLastCompactSize();
             if (ss != null && ss.getSize() > maxSize) {
               maxSize = ss.getSize();
               splitRow = ss.getSplitRow();
@@ -853,7 +870,10 @@ public class HRegion implements HeapSize
           LOG.info(((completed) ? "completed" : "aborted")
               + " compaction on region " + this
               + " after " + StringUtils.formatTimeDiff(now, startTime));
-          this.lastCompactPeriod = (completed) ? (now - startTime) / 1000 : 0;
+          if (completed) {
+            this.lastCompactInfo =
+              new Pair<Long,Long>((now - startTime) / 1000, lastCompactSize);
+          }
         }
       } finally {
         synchronized (writestate) {
@@ -1100,14 +1120,16 @@ public class HRegion implements HeapSize
       notifyAll(); // FindBugs NN_NAKED_NOTIFY
     }
 
+    long time = EnvironmentEdgeManager.currentTimeMillis() - startTime;
     if (LOG.isDebugEnabled()) {
-      long now = EnvironmentEdgeManager.currentTimeMillis();
       LOG.info("Finished memstore flush of ~" +
         StringUtils.humanReadableInt(currentMemStoreSize) + " for region " +
-        this + " in " + (now - startTime) + "ms, sequence id=" + sequenceId +
+        this + " in " + time + "ms, sequence id=" + sequenceId +
         ", compaction requested=" + compactionRequested +
         ((wal == null)? "; wal=null": ""));
     }
+    this.recentFlushes.add(new Pair<Long,Long>(time/1000,currentMemStoreSize));
+
     return compactionRequested;
   }
 
@@ -3118,7 +3140,7 @@ public class HRegion implements HeapSize
 
   public static final long FIXED_OVERHEAD = ClassSize.align(
       (4 * Bytes.SIZEOF_LONG) + Bytes.SIZEOF_BOOLEAN +
-      (21 * ClassSize.REFERENCE) + ClassSize.OBJECT + Bytes.SIZEOF_INT);
+      (22 * ClassSize.REFERENCE) + ClassSize.OBJECT + Bytes.SIZEOF_INT);
 
   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD +
       ClassSize.OBJECT + (2 * ClassSize.ATOMIC_BOOLEAN) +

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStoreFlusher.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStoreFlusher.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStoreFlusher.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStoreFlusher.java Tue Oct 11 02:06:21 2011
@@ -250,6 +250,7 @@ class MemStoreFlusher extends Thread imp
       if (region.flushcache()) {
         server.compactSplitThread.requestCompaction(region, getName());
       }
+      server.getMetrics().addFlush(region.getRecentFlushInfo());
     } catch (DroppedSnapshotException ex) {
       // Cache flush can fail in a few places. If it fails in a critical
       // section, we get a DroppedSnapshotException and a replay of hlog

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java Tue Oct 11 02:06:21 2011
@@ -94,6 +94,7 @@ public class Store implements HeapSize {
   protected long ttl;
   private long majorCompactionTime;
   private int maxFilesToCompact;
+  private long lastCompactSize = 0;
   /* how many bytes to write between status checks */
   static int closeCheckInterval = 0;
   private final long desiredMaxFileSize;
@@ -586,6 +587,8 @@ public class Store implements HeapSize {
     boolean forceSplit = this.region.shouldSplit(false);
     boolean majorcompaction = mc;
     synchronized (compactLock) {
+      this.lastCompactSize = 0;
+
       // filesToCompact are sorted oldest to newest.
       List<StoreFile> filesToCompact = this.storefiles;
       if (filesToCompact.isEmpty()) {
@@ -669,6 +672,7 @@ public class Store implements HeapSize {
             " file(s), size: " + skipped);
         }
       }
+      this.lastCompactSize = totalSize - skipped;
 
       // Ready to go.  Have list of files to compact.
       LOG.info("Started compaction of " + filesToCompact.size() + " file(s) in " +
@@ -1236,6 +1240,11 @@ public class Store implements HeapSize {
     return null;
   }
 
+  /** @return aggregate size of all HStores used in the last compaction */
+  public long getLastCompactSize() {
+    return this.lastCompactSize;
+  }
+
   /** @return aggregate size of HStore */
   public long getSize() {
     return storeSize;
@@ -1415,7 +1424,7 @@ public class Store implements HeapSize {
 
   public static final long FIXED_OVERHEAD = ClassSize.align(
       ClassSize.OBJECT + (14 * ClassSize.REFERENCE) +
-      (4 * Bytes.SIZEOF_LONG) + (3 * Bytes.SIZEOF_INT) + (Bytes.SIZEOF_BOOLEAN * 2));
+      (5 * Bytes.SIZEOF_LONG) + (3 * Bytes.SIZEOF_INT) + (Bytes.SIZEOF_BOOLEAN * 2));
 
   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD +
       ClassSize.OBJECT + ClassSize.REENTRANT_LOCK +

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/metrics/RegionServerMetrics.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/metrics/RegionServerMetrics.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/metrics/RegionServerMetrics.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/metrics/RegionServerMetrics.java Tue Oct 11 02:06:21 2011
@@ -23,7 +23,9 @@ import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.hbase.io.hfile.HFile;
 import org.apache.hadoop.hbase.metrics.MetricsRate;
+import org.apache.hadoop.hbase.metrics.PersistentMetricsTimeVaryingRate;
 import org.apache.hadoop.hbase.regionserver.wal.HLog;
+import org.apache.hadoop.hbase.util.Pair;
 import org.apache.hadoop.hbase.util.Strings;
 import org.apache.hadoop.metrics.ContextFactory;
 import org.apache.hadoop.metrics.MetricsContext;
@@ -39,6 +41,8 @@ import org.apache.hadoop.metrics.util.Me
 import java.io.IOException;
 import java.lang.management.ManagementFactory;
 import java.lang.management.MemoryUsage;
+import java.util.ArrayList;
+import java.util.List;
 
 /**
  * This class is for maintaining the various regionserver statistics
@@ -52,8 +56,8 @@ public class RegionServerMetrics impleme
   private final Log LOG = LogFactory.getLog(this.getClass());
   private final MetricsRecord metricsRecord;
   private long lastUpdate = System.currentTimeMillis();
-  private long lastMarathon = System.currentTimeMillis();
-  private long marathonPeriod = 0;
+  private long lastExtUpdate = System.currentTimeMillis();
+  private long extendedPeriod = 0;
   private static final int MB = 1024*1024;
   private MetricsRegistry registry = new MetricsRegistry();
   private final RegionServerStatistics statistics;
@@ -141,8 +145,20 @@ public class RegionServerMetrics impleme
   /**
    * time each scheduled compaction takes
    */
-  public final MetricsTimeVaryingRate compaction =
-    new MetricsTimeVaryingRate("compaction", registry);
+  protected final PersistentMetricsTimeVaryingRate compactionTime =
+    new PersistentMetricsTimeVaryingRate("compactionTime", registry);
+
+  protected final PersistentMetricsTimeVaryingRate compactionSize =
+    new PersistentMetricsTimeVaryingRate("compactionSize", registry);
+
+  /**
+   * time each scheduled flush takes
+   */
+  protected final PersistentMetricsTimeVaryingRate flushTime =
+    new PersistentMetricsTimeVaryingRate("flushTime", registry);
+
+  protected final PersistentMetricsTimeVaryingRate flushSize =
+    new PersistentMetricsTimeVaryingRate("flushSize", registry);
 
   public RegionServerMetrics() {
     MetricsContext context = MetricsUtil.getContext("hbase");
@@ -158,9 +174,9 @@ public class RegionServerMetrics impleme
 
     // get custom attributes
     try {
-      Object m = ContextFactory.getFactory().getAttribute("hbase.marathon");
+      Object m = ContextFactory.getFactory().getAttribute("hbase.extendedperiod");
       if (m instanceof String) {
-        this.marathonPeriod = Long.parseLong((String) m)*1000;
+        this.extendedPeriod = Long.parseLong((String) m)*1000;
       }
     } catch (IOException ioe) {
       LOG.info("Couldn't load ContextFactory for Metrics config info");
@@ -183,11 +199,14 @@ public class RegionServerMetrics impleme
     synchronized (this) {
       this.lastUpdate = System.currentTimeMillis();
 
-      // has the marathon period for long-living stats elapsed?
-      boolean marathonExpired = (this.marathonPeriod > 0) &&
-          (this.lastUpdate - this.lastMarathon >= this.marathonPeriod);
-      if (marathonExpired) {
-        this.lastMarathon = this.lastUpdate;
+      // has the extended period for long-living stats elapsed?
+      if (this.extendedPeriod > 0 &&
+          this.lastUpdate - this.lastExtUpdate >= this.extendedPeriod) {
+        this.lastExtUpdate = this.lastUpdate;
+        this.compactionTime.resetMinMaxAvg();
+        this.compactionSize.resetMinMaxAvg();
+        this.flushTime.resetMinMaxAvg();
+        this.flushSize.resetMinMaxAvg();
         this.resetAllMinMax();
       }
 
@@ -226,39 +245,19 @@ public class RegionServerMetrics impleme
       this.fsReadLatency.pushMetric(this.metricsRecord);
       this.fsWriteLatency.pushMetric(this.metricsRecord);
       this.fsSyncLatency.pushMetric(this.metricsRecord);
-      this.compaction.pushMetric(this.metricsRecord);
-
-      if (!marathonExpired) {
-        maintainStats(this.compaction);
-      }
+      this.compactionTime.pushMetric(this.metricsRecord);
+      this.compactionSize.pushMetric(this.metricsRecord);
+      this.flushTime.pushMetric(this.metricsRecord);
+      this.flushSize.pushMetric(this.metricsRecord);
     }
     this.metricsRecord.update();
   }
 
-  /* MetricsTimeVaryingRate will reset every time pushMetric() is called
-   * This is annoying for long-running stats that might not get a single
-   * operation in the polling period.  This function ensures that values
-   * for those stat entries don't get reset.
-   */
-  protected void maintainStats(MetricsTimeVaryingRate stat) {
-    int curOps = stat.getPreviousIntervalNumOps();
-    if (curOps > 0) {
-      long curTime = stat.getPreviousIntervalAverageTime();
-      long totalTime = curTime * curOps;
-      if (totalTime / curTime == curOps) {
-        stat.inc(curOps, totalTime);
-      } else {
-        LOG.info("Stats for " + stat.getName() + " overflowed! resetting");
-      }
-    }
-  }
-
   public void resetAllMinMax() {
     this.atomicIncrementTime.resetMinMax();
     this.fsReadLatency.resetMinMax();
     this.fsWriteLatency.resetMinMax();
     this.fsSyncLatency.resetMinMax();
-    this.compaction.resetMinMax();
   }
 
   /**
@@ -269,11 +268,20 @@ public class RegionServerMetrics impleme
   }
 
   /**
-   * @param periodSec seconds that last compaction took
+   * @param compact history in <time, size>
    */
-  public void addCompaction(final long periodSec) {
-    synchronized (this) {
-      this.compaction.inc(periodSec);
+  public synchronized void addCompaction(final Pair<Long,Long> compact) {
+    this.compactionTime.inc(compact.getFirst());
+    this.compactionSize.inc(compact.getSecond());
+  }
+
+  /**
+   * @param flushes history in <time, size>
+   */
+  public synchronized void addFlush(final List<Pair<Long,Long>> flushes) {
+    for (Pair<Long,Long> f : flushes) {
+      this.flushTime.inc(f.getFirst());
+      this.flushSize.inc(f.getSecond());
     }
   }
 

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/wal/HLog.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/wal/HLog.java?rev=1181413&r1=1181412&r2=1181413&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/wal/HLog.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/wal/HLog.java Tue Oct 11 02:06:21 2011
@@ -254,6 +254,9 @@ public class HLog implements Syncable {
   private static volatile long syncOps;
   private static volatile long syncTime;
 
+  public static volatile long lastSplitTime = 0;
+  public static volatile long lastSplitSize = 0;
+
   public static long getWriteOps() {
     long ret = writeOps;
     writeOps = 0;
@@ -1187,7 +1190,7 @@ public class HLog implements Syncable {
     Path oldLogDir, final FileSystem fs, final Configuration conf)
   throws IOException {
 
-    long millis = System.currentTimeMillis();
+    long startTime = System.currentTimeMillis();
     List<Path> splits = null;
     if (!fs.exists(srcDir)) {
       // Nothing to do
@@ -1218,9 +1221,9 @@ public class HLog implements Syncable {
       io.initCause(e);
       throw io;
     }
-    long endMillis = System.currentTimeMillis();
-    LOG.info("hlog file splitting completed in " + (endMillis - millis) +
-        " millis for " + srcDir.toString());
+    lastSplitTime = System.currentTimeMillis() - startTime;
+    LOG.info("hlog file splitting completed in " + lastSplitTime +
+        " ms for " + srcDir.toString());
     return splits;
   }
 
@@ -1293,8 +1296,9 @@ public class HLog implements Syncable {
     // More means faster but bigger mem consumption
     //TODO make a note on the conf rename and update hbase-site.xml if needed
     int logFilesPerStep = conf.getInt("hbase.hlog.split.batch.size", 3);
-     boolean skipErrors = conf.getBoolean("hbase.hlog.split.skip.errors", false);
+    boolean skipErrors = conf.getBoolean("hbase.hlog.split.skip.errors", false);
 
+    lastSplitSize = 0;
 
     try {
       int i = -1;
@@ -1309,6 +1313,7 @@ public class HLog implements Syncable {
           FileStatus log = logfiles[i];
           Path logPath = log.getPath();
           long logLength = log.getLen();
+          lastSplitSize += logLength;
           LOG.debug("Splitting hlog " + (i + 1) + " of " + logfiles.length +
             ": " + logPath + ", length=" + logLength );
           try {