001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.regionserver;
019
020import java.io.IOException;
021import org.apache.hadoop.hbase.HBaseClassTestRule;
022import org.apache.hadoop.hbase.HBaseTestingUtility;
023import org.apache.hadoop.hbase.HConstants;
024import org.apache.hadoop.hbase.HRegionInfo;
025import org.apache.hadoop.hbase.NotServingRegionException;
026import org.apache.hadoop.hbase.ServerName;
027import org.apache.hadoop.hbase.TableName;
028import org.apache.hadoop.hbase.client.Put;
029import org.apache.hadoop.hbase.client.RegionInfo;
030import org.apache.hadoop.hbase.client.RegionLocator;
031import org.apache.hadoop.hbase.client.Table;
032import org.apache.hadoop.hbase.client.TableDescriptor;
033import org.apache.hadoop.hbase.master.HMaster;
034import org.apache.hadoop.hbase.regionserver.handler.OpenRegionHandler;
035import org.apache.hadoop.hbase.testclassification.MediumTests;
036import org.apache.hadoop.hbase.testclassification.RegionServerTests;
037import org.apache.hadoop.hbase.util.Bytes;
038import org.apache.hadoop.hbase.util.JVMClusterUtil;
039import org.apache.hadoop.hbase.util.JVMClusterUtil.RegionServerThread;
040import org.apache.hadoop.hbase.util.Threads;
041import org.junit.AfterClass;
042import org.junit.Assert;
043import org.junit.BeforeClass;
044import org.junit.ClassRule;
045import org.junit.Test;
046import org.junit.experimental.categories.Category;
047import org.slf4j.Logger;
048import org.slf4j.LoggerFactory;
049
050import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
051import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
052import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos;
053import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.CloseRegionRequest;
054
055/**
056 * Tests on the region server, without the master.
057 */
058@Category({ RegionServerTests.class, MediumTests.class })
059public class TestRegionServerNoMaster {
060
061  @ClassRule
062  public static final HBaseClassTestRule CLASS_RULE =
063    HBaseClassTestRule.forClass(TestRegionServerNoMaster.class);
064
065  private static final Logger LOG = LoggerFactory.getLogger(TestRegionServerNoMaster.class);
066  private static final int NB_SERVERS = 1;
067  private static Table table;
068  private static final byte[] row = Bytes.toBytes("ee");
069
070  private static HRegionInfo hri;
071
072  private static byte[] regionName;
073  private static final HBaseTestingUtility HTU = new HBaseTestingUtility();
074
075  @BeforeClass
076  public static void before() throws Exception {
077    HTU.startMiniCluster(NB_SERVERS);
078    final TableName tableName = TableName.valueOf(TestRegionServerNoMaster.class.getSimpleName());
079
080    // Create table then get the single region for our new table.
081    table = HTU.createTable(tableName, HConstants.CATALOG_FAMILY);
082    Put p = new Put(row);
083    p.addColumn(HConstants.CATALOG_FAMILY, row, row);
084    table.put(p);
085
086    try (RegionLocator locator = HTU.getConnection().getRegionLocator(tableName)) {
087      hri = locator.getRegionLocation(row, false).getRegionInfo();
088    }
089    regionName = hri.getRegionName();
090
091    stopMasterAndCacheMetaLocation(HTU);
092  }
093
094  public static void stopMasterAndCacheMetaLocation(HBaseTestingUtility HTU)
095    throws IOException, InterruptedException {
096    // cache meta location, so we will not go to master to lookup meta region location
097    for (JVMClusterUtil.RegionServerThread t : HTU.getMiniHBaseCluster().getRegionServerThreads()) {
098      try (RegionLocator locator =
099        t.getRegionServer().getConnection().getRegionLocator(TableName.META_TABLE_NAME)) {
100        locator.getAllRegionLocations();
101      }
102    }
103    try (RegionLocator locator = HTU.getConnection().getRegionLocator(TableName.META_TABLE_NAME)) {
104      locator.getAllRegionLocations();
105    }
106    // Stop master
107    HMaster master = HTU.getHBaseCluster().getMaster();
108    Thread masterThread = HTU.getHBaseCluster().getMasterThread();
109    master.stopMaster();
110
111    LOG.info("Waiting until master thread exits");
112    while (masterThread != null && masterThread.isAlive()) {
113      Threads.sleep(100);
114    }
115
116    HRegionServer.TEST_SKIP_REPORTING_TRANSITION = true;
117  }
118
119  /**
120   * Flush the given region in the mini cluster. Since no master, we cannot use HBaseAdmin.flush()
121   */
122  public static void flushRegion(HBaseTestingUtility HTU, RegionInfo regionInfo)
123    throws IOException {
124    for (RegionServerThread rst : HTU.getMiniHBaseCluster().getRegionServerThreads()) {
125      HRegion region = rst.getRegionServer().getRegionByEncodedName(regionInfo.getEncodedName());
126      if (region != null) {
127        region.flush(true);
128        return;
129      }
130    }
131    throw new IOException("Region to flush cannot be found");
132  }
133
134  @AfterClass
135  public static void afterClass() throws Exception {
136    HRegionServer.TEST_SKIP_REPORTING_TRANSITION = false;
137    if (table != null) {
138      table.close();
139    }
140    HTU.shutdownMiniCluster();
141  }
142
143  private static HRegionServer getRS() {
144    return HTU.getHBaseCluster().getLiveRegionServerThreads().get(0).getRegionServer();
145  }
146
147  public static void openRegion(HBaseTestingUtility HTU, HRegionServer rs, HRegionInfo hri)
148    throws Exception {
149    AdminProtos.OpenRegionRequest orr =
150      RequestConverter.buildOpenRegionRequest(rs.getServerName(), hri, null);
151    AdminProtos.OpenRegionResponse responseOpen = rs.rpcServices.openRegion(null, orr);
152
153    Assert.assertTrue(responseOpen.getOpeningStateCount() == 1);
154    Assert.assertTrue(responseOpen.getOpeningState(0)
155      .equals(AdminProtos.OpenRegionResponse.RegionOpeningState.OPENED));
156
157    checkRegionIsOpened(HTU, rs, hri);
158  }
159
160  public static void checkRegionIsOpened(HBaseTestingUtility HTU, HRegionServer rs, HRegionInfo hri)
161    throws Exception {
162    while (!rs.getRegionsInTransitionInRS().isEmpty()) {
163      Thread.sleep(1);
164    }
165
166    Assert.assertTrue(rs.getRegion(hri.getRegionName()).isAvailable());
167  }
168
169  public static void closeRegion(HBaseTestingUtility HTU, HRegionServer rs, HRegionInfo hri)
170    throws Exception {
171    AdminProtos.CloseRegionRequest crr =
172      ProtobufUtil.buildCloseRegionRequest(rs.getServerName(), hri.getRegionName());
173    AdminProtos.CloseRegionResponse responseClose = rs.rpcServices.closeRegion(null, crr);
174    Assert.assertTrue(responseClose.getClosed());
175    checkRegionIsClosed(HTU, rs, hri);
176  }
177
178  public static void checkRegionIsClosed(HBaseTestingUtility HTU, HRegionServer rs, HRegionInfo hri)
179    throws Exception {
180    while (!rs.getRegionsInTransitionInRS().isEmpty()) {
181      Thread.sleep(1);
182    }
183
184    try {
185      Assert.assertFalse(rs.getRegion(hri.getRegionName()).isAvailable());
186    } catch (NotServingRegionException expected) {
187      // That's how it work: if the region is closed we have an exception.
188    }
189  }
190
191  /**
192   * Close the region without using ZK
193   */
194  private void closeRegionNoZK() throws Exception {
195    // no transition in ZK
196    AdminProtos.CloseRegionRequest crr =
197      ProtobufUtil.buildCloseRegionRequest(getRS().getServerName(), regionName);
198    AdminProtos.CloseRegionResponse responseClose = getRS().rpcServices.closeRegion(null, crr);
199    Assert.assertTrue(responseClose.getClosed());
200
201    // now waiting & checking. After a while, the transition should be done and the region closed
202    checkRegionIsClosed(HTU, getRS(), hri);
203  }
204
205  @Test
206  public void testCloseByRegionServer() throws Exception {
207    closeRegionNoZK();
208    openRegion(HTU, getRS(), hri);
209  }
210
211  @Test
212  public void testMultipleCloseFromMaster() throws Exception {
213    for (int i = 0; i < 10; i++) {
214      AdminProtos.CloseRegionRequest crr =
215        ProtobufUtil.buildCloseRegionRequest(getRS().getServerName(), regionName, null);
216      try {
217        AdminProtos.CloseRegionResponse responseClose = getRS().rpcServices.closeRegion(null, crr);
218        Assert.assertTrue("request " + i + " failed",
219          responseClose.getClosed() || responseClose.hasClosed());
220      } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException se) {
221        Assert.assertTrue("The next queries may throw an exception.", i > 0);
222      }
223    }
224
225    checkRegionIsClosed(HTU, getRS(), hri);
226
227    openRegion(HTU, getRS(), hri);
228  }
229
230  /**
231   * Test that if we do a close while opening it stops the opening.
232   */
233  @Test
234  public void testCancelOpeningWithoutZK() throws Exception {
235    // We close
236    closeRegionNoZK();
237    checkRegionIsClosed(HTU, getRS(), hri);
238
239    // Let do the initial steps, without having a handler
240    getRS().getRegionsInTransitionInRS().put(hri.getEncodedNameAsBytes(), Boolean.TRUE);
241
242    // That's a close without ZK.
243    AdminProtos.CloseRegionRequest crr =
244      ProtobufUtil.buildCloseRegionRequest(getRS().getServerName(), regionName);
245    try {
246      getRS().rpcServices.closeRegion(null, crr);
247      Assert.assertTrue(false);
248    } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException expected) {
249    }
250
251    // The state in RIT should have changed to close
252    Assert.assertEquals(Boolean.FALSE,
253      getRS().getRegionsInTransitionInRS().get(hri.getEncodedNameAsBytes()));
254
255    // Let's start the open handler
256    TableDescriptor htd = getRS().tableDescriptors.get(hri.getTable());
257
258    getRS().executorService.submit(new OpenRegionHandler(getRS(), getRS(), hri, htd, -1));
259
260    // The open handler should have removed the region from RIT but kept the region closed
261    checkRegionIsClosed(HTU, getRS(), hri);
262
263    openRegion(HTU, getRS(), hri);
264  }
265
266  /**
267   * Tests an on-the-fly RPC that was scheduled for the earlier RS on the same port for openRegion.
268   * The region server should reject this RPC. (HBASE-9721)
269   */
270  @Test
271  public void testOpenCloseRegionRPCIntendedForPreviousServer() throws Exception {
272    Assert.assertTrue(getRS().getRegion(regionName).isAvailable());
273
274    ServerName sn = getRS().getServerName();
275    ServerName earlierServerName = ServerName.valueOf(sn.getHostname(), sn.getPort(), 1);
276
277    try {
278      CloseRegionRequest request =
279        ProtobufUtil.buildCloseRegionRequest(earlierServerName, regionName);
280      getRS().getRSRpcServices().closeRegion(null, request);
281      Assert.fail("The closeRegion should have been rejected");
282    } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException se) {
283      Assert.assertTrue(se.getCause() instanceof IOException);
284      Assert.assertTrue(
285        se.getCause().getMessage().contains("This RPC was intended for a different server"));
286    }
287
288    // actual close
289    closeRegionNoZK();
290    try {
291      AdminProtos.OpenRegionRequest orr =
292        RequestConverter.buildOpenRegionRequest(earlierServerName, hri, null);
293      getRS().getRSRpcServices().openRegion(null, orr);
294      Assert.fail("The openRegion should have been rejected");
295    } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException se) {
296      Assert.assertTrue(se.getCause() instanceof IOException);
297      Assert.assertTrue(
298        se.getCause().getMessage().contains("This RPC was intended for a different server"));
299    } finally {
300      openRegion(HTU, getRS(), hri);
301    }
302  }
303}