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 2010/12/28 00:38:27 UTC

svn commit: r1053225 - in /hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver: HRegion.java Store.java

Author: nspiegelberg
Date: Mon Dec 27 23:38:27 2010
New Revision: 1053225

URL: http://svn.apache.org/viewvc?rev=1053225&view=rev
Log:
HBASE-3384 : Move User-Triggered Compactions to Store

Modified:
    hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
    hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java

Modified: hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
URL: http://svn.apache.org/viewvc/hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java?rev=1053225&r1=1053224&r2=1053225&view=diff
==============================================================================
--- hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java (original)
+++ hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java Mon Dec 27 23:38:27 2010
@@ -184,11 +184,6 @@ public class HRegion implements HeapSize
   final Path regiondir;
   KeyValue.KVComparator comparator;
 
-  /*
-   * Set this when scheduling compaction if want the next compaction to be a
-   * major compaction.  Cleared each time through compaction code.
-   */
-  private volatile boolean forceMajorCompaction = false;
   private Pair<Long,Long> lastCompactInfo = null;
 
   /*
@@ -716,11 +711,9 @@ public class HRegion implements HeapSize
   }
 
   void setForceMajorCompaction(final boolean b) {
-    this.forceMajorCompaction = b;
-  }
-
-  boolean getForceMajorCompaction() {
-    return this.forceMajorCompaction;
+    for (Store h: stores.values()) {
+      h.setForceMajorCompaction(b);
+    }
   }
 
   /**
@@ -734,13 +727,14 @@ public class HRegion implements HeapSize
    * conflicts with a region split, and that cannot happen because the region
    * server does them sequentially and not in parallel.
    *
-   * @return mid key if split is needed
+   * @param majorCompaction True to force a major compaction regardless of thresholds
+   * @return split row if split is needed
    * @throws IOException e
    */
-  public byte [] compactStores() throws IOException {
-    boolean majorCompaction = this.forceMajorCompaction;
-    this.forceMajorCompaction = false;
-    return compactStores(majorCompaction);
+  byte [] compactStores(final boolean majorCompaction)
+  throws IOException {
+    this.setForceMajorCompaction(majorCompaction);
+    return compactStores();
   }
 
   /*
@@ -754,12 +748,10 @@ public class HRegion implements HeapSize
    * conflicts with a region split, and that cannot happen because the region
    * server does them sequentially and not in parallel.
    *
-   * @param majorCompaction True to force a major compaction regardless of thresholds
    * @return split row if split is needed
    * @throws IOException e
    */
-  byte [] compactStores(final boolean majorCompaction)
-  throws IOException {
+  public byte [] compactStores() throws IOException {
     if (this.closing.get()) {
       LOG.debug("Skipping compaction on " + this + " because closing");
       return null;
@@ -789,8 +781,7 @@ public class HRegion implements HeapSize
               return splitRow;
           }
         }
-        LOG.info("Starting" + (majorCompaction? " major " : " ") +
-            "compaction on region " + this);
+        LOG.info("Starting compaction on region " + this);
         long startTime = EnvironmentEdgeManager.currentTimeMillis();
         doRegionCompactionPrep();
         long lastCompactSize = 0;
@@ -798,7 +789,7 @@ public class HRegion implements HeapSize
         boolean completed = false;
         try {
           for (Store store: stores.values()) {
-            final Store.StoreSize ss = store.compact(majorCompaction);
+            final Store.StoreSize ss = store.compact();
             lastCompactSize += store.getLastCompactSize();
             if (ss != null && ss.getSize() > maxSize) {
               maxSize = ss.getSize();
@@ -3320,7 +3311,7 @@ public class HRegion implements HeapSize
   }
 
   public static final long FIXED_OVERHEAD = ClassSize.align(
-      (4 * Bytes.SIZEOF_LONG) + Bytes.SIZEOF_BOOLEAN + ClassSize.ARRAY +
+      (4 * Bytes.SIZEOF_LONG) + ClassSize.ARRAY +
       (24 * ClassSize.REFERENCE) + ClassSize.OBJECT + Bytes.SIZEOF_INT);
 
   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD +

Modified: hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java
URL: http://svn.apache.org/viewvc/hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java?rev=1053225&r1=1053224&r2=1053225&view=diff
==============================================================================
--- hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java (original)
+++ hbase/trunk/src/main/java/org/apache/hadoop/hbase/regionserver/Store.java Mon Dec 27 23:38:27 2010
@@ -101,6 +101,7 @@ public class Store implements HeapSize {
   // With float, java will downcast your long to float for comparisons (bad)
   private double compactRatio;
   private long lastCompactSize = 0;
+  private volatile boolean forceMajor = false;
   /* how many bytes to write between status checks */
   static int closeCheckInterval = 0;
   private final long desiredMaxFileSize;
@@ -609,11 +610,10 @@ public class Store implements HeapSize {
    * <p>We don't want to hold the structureLock for the whole time, as a compact()
    * can be lengthy and we want to allow cache-flushes during this period.
    *
-   * @param forceMajor True to force a major compaction regardless of thresholds
    * @return row to split around if a split is needed, null otherwise
    * @throws IOException
    */
-  StoreSize compact(final boolean forceMajor) throws IOException {
+  StoreSize compact() throws IOException {
     boolean forceSplit = this.region.shouldForceSplit();
     synchronized (compactLock) {
       this.lastCompactSize = 0; // reset first in case compaction is aborted
@@ -629,12 +629,12 @@ public class Store implements HeapSize {
 
       // if the user wants to force a split, skip compaction unless necessary
       boolean references = hasReferences(this.storefiles);
-      if (forceSplit && !forceMajor && !references) {
+      if (forceSplit && !this.forceMajor && !references) {
         return checkSplit(forceSplit);
       }
 
       Collection<StoreFile> filesToCompact
-        = compactSelection(this.storefiles, forceMajor);
+        = compactSelection(this.storefiles, this.forceMajor);
 
       // empty == do not compact
       if (filesToCompact.isEmpty()) {
@@ -652,6 +652,9 @@ public class Store implements HeapSize {
       // major compaction iff all StoreFiles are included
       boolean majorcompaction
         = (filesToCompact.size() == this.storefiles.size());
+      if (majorcompaction) {
+        this.forceMajor = false;
+      }
 
       // Max-sequenceID is the last key in the files we're compacting
       long maxId = StoreFile.getMaxSequenceIdInList(filesToCompact);
@@ -1407,6 +1410,14 @@ public class Store implements HeapSize {
     return storeSize;
   }
 
+  void setForceMajorCompaction(final boolean b) {
+    this.forceMajor = b;
+  }
+
+  boolean getForceMajorCompaction() {
+    return this.forceMajor;
+  }
+
   //////////////////////////////////////////////////////////////////////////////
   // File administration
   //////////////////////////////////////////////////////////////////////////////
@@ -1613,7 +1624,7 @@ public class Store implements HeapSize {
   public static final long FIXED_OVERHEAD = ClassSize.align(
       ClassSize.OBJECT + (15 * ClassSize.REFERENCE) +
       (7 * Bytes.SIZEOF_LONG) + (1 * Bytes.SIZEOF_DOUBLE) +
-      (4 * Bytes.SIZEOF_INT) + (Bytes.SIZEOF_BOOLEAN * 2));
+      (4 * Bytes.SIZEOF_INT) + (3 * Bytes.SIZEOF_BOOLEAN));
 
   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD +
       ClassSize.OBJECT + ClassSize.REENTRANT_LOCK +