You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by jd...@apache.org on 2010/05/11 02:17:35 UTC

svn commit: r942962 - in /hadoop/hbase/trunk: ./ core/src/main/java/org/apache/hadoop/hbase/master/ core/src/main/java/org/apache/hadoop/hbase/regionserver/ core/src/test/java/org/apache/hadoop/hbase/master/

Author: jdcryans
Date: Tue May 11 00:17:35 2010
New Revision: 942962

URL: http://svn.apache.org/viewvc?rev=942962&view=rev
Log:
HBASE-2515  ChangeTableState considers split&&offline regions as being served
HBASE-2528  ServerManager.ServerMonitor isn't daemonized

Added:
    hadoop/hbase/trunk/core/src/test/java/org/apache/hadoop/hbase/master/TestMasterWithDisabling.java
Modified:
    hadoop/hbase/trunk/CHANGES.txt
    hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/master/ServerManager.java
    hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java

Modified: hadoop/hbase/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/hadoop/hbase/trunk/CHANGES.txt?rev=942962&r1=942961&r2=942962&view=diff
==============================================================================
--- hadoop/hbase/trunk/CHANGES.txt (original)
+++ hadoop/hbase/trunk/CHANGES.txt Tue May 11 00:17:35 2010
@@ -306,6 +306,7 @@ Release 0.21.0 - Unreleased
    HBASE-2503  PriorityQueue isn't thread safe, KeyValueHeap uses it that way
    HBASE-2431  Master does not respect generation stamps, may result in meta
                getting permanently offlined
+   HBASE-2515  ChangeTableState considers split&&offline regions as being served
 
   IMPROVEMENTS
    HBASE-1760  Cleanup TODOs in HTable
@@ -554,6 +555,7 @@ Release 0.21.0 - Unreleased
    HBASE-2414  Enhance test suite to be able to specify distributed scenarios
    HBASE-2518  Kill all the trailing whitespaces in the code base
                (Benoit Sigoure via Stack)
+   HBASE-2528  ServerManager.ServerMonitor isn't daemonized
 
   NEW FEATURES
    HBASE-1961  HBase EC2 scripts

Modified: hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/master/ServerManager.java
URL: http://svn.apache.org/viewvc/hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/master/ServerManager.java?rev=942962&r1=942961&r2=942962&view=diff
==============================================================================
--- hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/master/ServerManager.java (original)
+++ hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/master/ServerManager.java Tue May 11 00:17:35 2010
@@ -148,13 +148,15 @@ public class ServerManager implements HC
     this.minimumServerCount = c.getInt("hbase.regions.server.count.min", 0);
     this.serverMonitorThread = new ServerMonitor(metaRescanInterval,
       this.master.getShutdownRequested());
-    this.serverMonitorThread.start();
+    String n = Thread.currentThread().getName();
+    Threads.setDaemonThreadRunning(this.serverMonitorThread,
+      n + ".serverMonitor");
     this.oldLogCleaner = new OldLogsCleaner(
       c.getInt("hbase.master.meta.thread.rescanfrequency",60 * 1000),
         this.master.getShutdownRequested(), c,
         master.getFileSystem(), master.getOldLogDir());
     Threads.setDaemonThreadRunning(oldLogCleaner,
-      "ServerManager.oldLogCleaner");
+      n + ".oldLogCleaner");
 
   }
 

Modified: hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java
URL: http://svn.apache.org/viewvc/hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java?rev=942962&r1=942961&r2=942962&view=diff
==============================================================================
--- hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java (original)
+++ hadoop/hbase/trunk/core/src/main/java/org/apache/hadoop/hbase/regionserver/CompactSplitThread.java Tue May 11 00:17:35 2010
@@ -183,6 +183,8 @@ class CompactSplitThread extends Thread 
     Put put = new Put(oldRegionInfo.getRegionName());
     put.add(CATALOG_FAMILY, REGIONINFO_QUALIFIER,
       Writables.getBytes(oldRegionInfo));
+    put.add(CATALOG_FAMILY, SERVER_QUALIFIER, EMPTY_BYTE_ARRAY);
+    put.add(CATALOG_FAMILY, STARTCODE_QUALIFIER, EMPTY_BYTE_ARRAY);
     put.add(CATALOG_FAMILY, SPLITA_QUALIFIER,
       Writables.getBytes(newRegions[0].getRegionInfo()));
     put.add(CATALOG_FAMILY, SPLITB_QUALIFIER,

Added: hadoop/hbase/trunk/core/src/test/java/org/apache/hadoop/hbase/master/TestMasterWithDisabling.java
URL: http://svn.apache.org/viewvc/hadoop/hbase/trunk/core/src/test/java/org/apache/hadoop/hbase/master/TestMasterWithDisabling.java?rev=942962&view=auto
==============================================================================
--- hadoop/hbase/trunk/core/src/test/java/org/apache/hadoop/hbase/master/TestMasterWithDisabling.java (added)
+++ hadoop/hbase/trunk/core/src/test/java/org/apache/hadoop/hbase/master/TestMasterWithDisabling.java Tue May 11 00:17:35 2010
@@ -0,0 +1,124 @@
+/**
+ * Copyright 2010 The Apache Software Foundation
+ *
+ * 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.master;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HColumnDescriptor;
+import org.apache.hadoop.hbase.HMsg;
+import org.apache.hadoop.hbase.HServerInfo;
+import org.apache.hadoop.hbase.HTableDescriptor;
+import org.apache.hadoop.hbase.MiniHBaseCluster;
+import org.apache.hadoop.hbase.client.HBaseAdmin;
+import org.apache.hadoop.hbase.client.HTable;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import static org.junit.Assert.fail;
+import static org.junit.Assert.assertEquals;
+
+import java.io.IOException;
+
+/**
+ * Disabling is tricky. This class tests how the Master behaves during those
+ */
+public class TestMasterWithDisabling {
+
+  private static final Log LOG = LogFactory.getLog(TestMasterWithDisabling.class);
+  private static final HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
+  private static final byte[] TABLENAME = Bytes.toBytes("disabling");
+  private static final byte[] FAMILYNAME = Bytes.toBytes("fam");
+
+  @BeforeClass
+  public static void beforeAllTests() throws Exception {
+    // Start a cluster of two regionservers.
+    TEST_UTIL.startMiniCluster(2);
+  }
+
+  @AfterClass
+  public static void afterAllTests() throws IOException {
+    TEST_UTIL.shutdownMiniCluster();
+  }
+
+  @Test
+  public void testDisableBetweenSplit() throws IOException {
+    // Table that splits like crazy
+    HTableDescriptor htd = new HTableDescriptor(TABLENAME);
+    htd.setMaxFileSize(1024);
+    htd.setMemStoreFlushSize(1024);
+    HColumnDescriptor hcd = new HColumnDescriptor(FAMILYNAME);
+    htd.addFamily(hcd);
+    TEST_UTIL.getHBaseAdmin().createTable(htd);
+    HTable t = new HTable(TEST_UTIL.getConfiguration(), TABLENAME);
+    HBase2515Listener list = new HBase2515Listener(TEST_UTIL.getHBaseAdmin());
+    MiniHBaseCluster cluster = TEST_UTIL.getHBaseCluster();
+    HMaster m = cluster.getMaster();
+    m.getRegionServerOperationQueue().
+      registerRegionServerOperationListener(list);
+    try {
+      TEST_UTIL.loadTable(t, FAMILYNAME);
+    } catch (IOException ex) {
+      // We disable the table during a split, we will end up here
+      LOG.info("Expected", ex);
+    }
+    // Check that there's no region in flight, HBASE-2515
+    assertEquals(0,cluster.getMaster().
+        getClusterStatus().getRegionsInTransition().size());
+  }
+
+  /**
+   * Simple listener that simulates a very long processing of a split. When
+   * we catch it, we first disable the table then let the processing go forward
+   */
+  static class HBase2515Listener implements RegionServerOperationListener {
+    HBaseAdmin admin;
+
+    public HBase2515Listener(HBaseAdmin admin) {
+      this.admin = admin;
+    }
+
+    @Override
+    public boolean process(HServerInfo serverInfo, HMsg incomingMsg) {
+      if (!incomingMsg.isType(HMsg.Type.MSG_REPORT_SPLIT_INCLUDES_DAUGHTERS)) {
+        return true;
+      }
+      try {
+        LOG.info("Disabling table");
+        admin.disableTable(TABLENAME);
+      } catch (IOException e) {
+        LOG.warn(e);
+        fail("Disable should always work");
+      }
+      return true;
+    }
+
+    @Override
+    public boolean process(RegionServerOperation op) throws IOException {
+      return true;
+    }
+
+    @Override
+    public void processed(RegionServerOperation op) {
+    }
+  }
+
+}