You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by mb...@apache.org on 2012/04/07 00:11:46 UTC

svn commit: r1310608 - in /hbase/branches/0.89-fb/src: main/java/org/apache/hadoop/hbase/regionserver/ test/java/org/apache/hadoop/hbase/regionserver/

Author: mbautin
Date: Fri Apr  6 22:11:46 2012
New Revision: 1310608

URL: http://svn.apache.org/viewvc?rev=1310608&view=rev
Log:
[master] Fix bug of closed region transitioning from CLOSED to CLOSING

Summary:
Fixed the situation of a transtion from CLOSED to CLOSING in znode for a region.
This happens when Master sends another Close Region RPC right before the Region Server closes the region.
The message gets there right after the close happens, and the regionserver changes the state from CLOSE to CLOSING.

Fixed this by ignoring any close region requests that are marked as already closed

Test Plan: mvn -Dtest=TestHRegionDoubleClose test

Reviewers: madhuvaidya, pkhemani, kranganathan, mbautin

Reviewed By: kranganathan

CC: kranganathan, hbase-eng@, kannan, mbautin

Differential Revision: https://phabricator.fb.com/D435024

Task ID: 982325

Added:
    hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionDoubleClose.java
Modified:
    hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/RSZookeeperUpdater.java
    hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionClose.java

Modified: hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/RSZookeeperUpdater.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/RSZookeeperUpdater.java?rev=1310608&r1=1310607&r2=1310608&view=diff
==============================================================================
--- hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/RSZookeeperUpdater.java (original)
+++ hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/RSZookeeperUpdater.java Fri Apr  6 22:11:46 2012
@@ -75,6 +75,20 @@ public class RSZookeeperUpdater {
         LOG.error(msg);
         throw new IOException(msg);
       }
+      // if the master sends a region close message right after we actually
+      //  finished closing the region (on the region server), we shouldn't move
+      //  from a closed state to a closing state, so log and throw to abort
+      if (rsData.getHbEvent() == HBaseEventType.RS2ZK_REGION_CLOSED) {
+        String msg = "ZNode " + regionZNode
+            + " is already closed. Master sent a close request for " +
+            rsData.getRsName() + " after regionserver " +
+            regionServerName + " already closed it";
+        LOG.warn(msg);
+
+        // since we already closed the region, we can just return to
+        // closeRegion() and nothing else should be executed
+        return;
+      }
       this.zkVersion = stat.getVersion();
     } else {
       // create the region node in the unassigned directory first

Modified: hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionClose.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionClose.java?rev=1310608&r1=1310607&r2=1310608&view=diff
==============================================================================
--- hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionClose.java (original)
+++ hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionClose.java Fri Apr  6 22:11:46 2012
@@ -1,3 +1,22 @@
+/**
+ * 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.regionserver;
 
 import org.apache.commons.logging.Log;
@@ -12,33 +31,27 @@ import org.apache.hadoop.hbase.util.Byte
 import org.apache.hadoop.hbase.util.Writables;
 import org.apache.hadoop.hbase.zookeeper.ZooKeeperWrapper;
 import org.apache.zookeeper.data.Stat;
-
-import static org.junit.Assert.*;
-
-import org.junit.AfterClass;
-import org.junit.BeforeClass;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNull;
+import org.junit.After;
+import org.junit.Before;
 import org.junit.Test;
 
 public class TestHRegionClose {
-  final Log LOG = LogFactory.getLog(getClass());
-  private final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
-  private static byte[][] FAMILIES = { Bytes.toBytes("f1"),
+  protected final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
+  protected static byte[][] FAMILIES = { Bytes.toBytes("f1"),
       Bytes.toBytes("f2"), Bytes.toBytes("f3"), Bytes.toBytes("f4") };
+  protected HRegionServer server;
+  protected ZooKeeperWrapper zkWrapper;
+  protected HRegionInfo regionInfo;
+  protected String regionZNode;
 
-  @BeforeClass
-  public static void setUpBeforeClass() throws Exception {
+  @Before
+  public void setUp() throws Exception {
     TEST_UTIL.startMiniCluster(3);
-  }
 
-  @AfterClass
-  public static void tearDownAfterClass() throws Exception {
-    TEST_UTIL.shutdownMiniCluster();
-  }
-
-  @Test
-  public void testCloseHRegion() throws Exception {
     // Build some data.
-    byte[] tableName = Bytes.toBytes("testCloseHRegion");
+    byte[] tableName = Bytes.toBytes(getClass().getSimpleName());
     TEST_UTIL.createTable(tableName, FAMILIES);
     HTable table = new HTable(TEST_UTIL.getConfiguration(), tableName);
     for (int i = 0; i < FAMILIES.length; i++) {
@@ -49,16 +62,27 @@ public class TestHRegionClose {
 
     // Pick a regionserver.
     Configuration conf = TEST_UTIL.getConfiguration();
-    HRegionServer server = TEST_UTIL.getHBaseCluster().getRegionServer(0);
+    server = TEST_UTIL.getHBaseCluster().getRegionServer(0);
 
     HRegion[] region = server.getOnlineRegionsAsArray();
-    HRegionInfo regionInfo = region[0].getRegionInfo();
+    regionInfo = region[0].getRegionInfo();
 
     // Some initializtion relevant to zk.
-    ZooKeeperWrapper zkWrapper = server.getZooKeeperWrapper();
-    String regionZNode = zkWrapper.getZNode(
+    zkWrapper = server.getZooKeeperWrapper();
+    regionZNode = zkWrapper.getZNode(
         zkWrapper.getRegionInTransitionZNode(), regionInfo.getEncodedName());
+  }
+
+  @After
+  public void tearDown() throws Exception {
+    server = null;
+    zkWrapper = null;
+    regionInfo = null;
+    regionZNode = null;
+    TEST_UTIL.shutdownMiniCluster();
+  }
 
+  protected void tryCloseRegion() throws Exception {
     server.closeRegion(regionInfo, true);
 
     byte[] data = zkWrapper.readZNode(regionZNode, new Stat());
@@ -69,4 +93,9 @@ public class TestHRegionClose {
     assertNull(server.getOnlineRegion(regionInfo.getRegionName()));
     assertEquals(HBaseEventType.RS2ZK_REGION_CLOSED, rsData.getHbEvent());
   }
+
+  @Test
+  public void mainTest() throws Exception {
+    tryCloseRegion();
+  }
 }

Added: hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionDoubleClose.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionDoubleClose.java?rev=1310608&view=auto
==============================================================================
--- hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionDoubleClose.java (added)
+++ hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionDoubleClose.java Fri Apr  6 22:11:46 2012
@@ -0,0 +1,40 @@
+/**
+ * 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.regionserver;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+
+/**
+ * Test that RegionServer ignores a close request from Master
+ * if it already successfully closed a region.  There was an
+ * issue where it was changing the ZooKeeper state from
+ * CLOSED to CLOSING
+ */
+public class TestHRegionDoubleClose extends TestHRegionClose {
+  private static final Log LOG = LogFactory.getLog(TestHRegionDoubleClose.class);
+  @Override
+  public void mainTest() throws Exception {
+    tryCloseRegion();
+    LOG.info("Trying to close the region again, to check that the RegionServer "
+        + "is idempotent. i.e. CLOSED -> CLOSING transition bug");
+    tryCloseRegion();
+  }
+}