HBASE-24033 Add ut for loading the corrupt recovered hfiles (#1322)
[hbase.git] / hbase-server / src / test / java / org / apache / hadoop / hbase / MiniHBaseCluster.java
blob53c590b74e1d860ad0a032a8995176800e5fb8f4
1 /**
3 * Licensed to the Apache Software Foundation (ASF) under one
4 * or more contributor license agreements. See the NOTICE file
5 * distributed with this work for additional information
6 * regarding copyright ownership. The ASF licenses this file
7 * to you under the Apache License, Version 2.0 (the
8 * "License"); you may not use this file except in compliance
9 * with the License. You may obtain a copy of the License at
11 * http://www.apache.org/licenses/LICENSE-2.0
13 * Unless required by applicable law or agreed to in writing, software
14 * distributed under the License is distributed on an "AS IS" BASIS,
15 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16 * See the License for the specific language governing permissions and
17 * limitations under the License.
19 package org.apache.hadoop.hbase;
21 import java.io.IOException;
22 import java.security.PrivilegedAction;
23 import java.util.ArrayList;
24 import java.util.HashSet;
25 import java.util.List;
26 import java.util.Set;
27 import org.apache.hadoop.conf.Configuration;
28 import org.apache.hadoop.fs.FileSystem;
29 import org.apache.hadoop.hbase.master.HMaster;
30 import org.apache.hadoop.hbase.regionserver.HRegion;
31 import org.apache.hadoop.hbase.regionserver.HRegion.FlushResult;
32 import org.apache.hadoop.hbase.regionserver.HRegionServer;
33 import org.apache.hadoop.hbase.regionserver.Region;
34 import org.apache.hadoop.hbase.security.User;
35 import org.apache.hadoop.hbase.test.MetricsAssertHelper;
36 import org.apache.hadoop.hbase.util.JVMClusterUtil;
37 import org.apache.hadoop.hbase.util.JVMClusterUtil.MasterThread;
38 import org.apache.hadoop.hbase.util.JVMClusterUtil.RegionServerThread;
39 import org.apache.hadoop.hbase.util.Threads;
40 import org.apache.yetus.audience.InterfaceAudience;
41 import org.slf4j.Logger;
42 import org.slf4j.LoggerFactory;
44 import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse;
46 /**
47 * This class creates a single process HBase cluster.
48 * each server. The master uses the 'default' FileSystem. The RegionServers,
49 * if we are running on DistributedFilesystem, create a FileSystem instance
50 * each and will close down their instance on the way out.
52 @InterfaceAudience.Public
53 public class MiniHBaseCluster extends HBaseCluster {
54 private static final Logger LOG = LoggerFactory.getLogger(MiniHBaseCluster.class.getName());
55 public LocalHBaseCluster hbaseCluster;
56 private static int index;
58 /**
59 * Start a MiniHBaseCluster.
60 * @param conf Configuration to be used for cluster
61 * @param numRegionServers initial number of region servers to start.
62 * @throws IOException
64 public MiniHBaseCluster(Configuration conf, int numRegionServers)
65 throws IOException, InterruptedException {
66 this(conf, 1, numRegionServers);
69 /**
70 * Start a MiniHBaseCluster.
71 * @param conf Configuration to be used for cluster
72 * @param numMasters initial number of masters to start.
73 * @param numRegionServers initial number of region servers to start.
74 * @throws IOException
76 public MiniHBaseCluster(Configuration conf, int numMasters, int numRegionServers)
77 throws IOException, InterruptedException {
78 this(conf, numMasters, numRegionServers, null, null);
81 /**
82 * Start a MiniHBaseCluster.
83 * @param conf Configuration to be used for cluster
84 * @param numMasters initial number of masters to start.
85 * @param numRegionServers initial number of region servers to start.
87 public MiniHBaseCluster(Configuration conf, int numMasters, int numRegionServers,
88 Class<? extends HMaster> masterClass,
89 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> regionserverClass)
90 throws IOException, InterruptedException {
91 this(conf, numMasters, 0, numRegionServers, null, masterClass, regionserverClass);
94 /**
95 * @param rsPorts Ports that RegionServer should use; pass ports if you want to test cluster
96 * restart where for sure the regionservers come up on same address+port (but
97 * just with different startcode); by default mini hbase clusters choose new
98 * arbitrary ports on each cluster start.
99 * @throws IOException
100 * @throws InterruptedException
102 public MiniHBaseCluster(Configuration conf, int numMasters, int numAlwaysStandByMasters,
103 int numRegionServers, List<Integer> rsPorts, Class<? extends HMaster> masterClass,
104 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> regionserverClass)
105 throws IOException, InterruptedException {
106 super(conf);
108 // Hadoop 2
109 CompatibilityFactory.getInstance(MetricsAssertHelper.class).init();
111 init(numMasters, numAlwaysStandByMasters, numRegionServers, rsPorts, masterClass,
112 regionserverClass);
113 this.initialClusterStatus = getClusterMetrics();
116 public Configuration getConfiguration() {
117 return this.conf;
121 * Subclass so can get at protected methods (none at moment). Also, creates
122 * a FileSystem instance per instantiation. Adds a shutdown own FileSystem
123 * on the way out. Shuts down own Filesystem only, not All filesystems as
124 * the FileSystem system exit hook does.
126 public static class MiniHBaseClusterRegionServer extends HRegionServer {
127 private Thread shutdownThread = null;
128 private User user = null;
130 * List of RegionServers killed so far. ServerName also comprises startCode of a server,
131 * so any restarted instances of the same server will have different ServerName and will not
132 * coincide with past dead ones. So there's no need to cleanup this list.
134 static Set<ServerName> killedServers = new HashSet<>();
136 public MiniHBaseClusterRegionServer(Configuration conf)
137 throws IOException, InterruptedException {
138 super(conf);
139 this.user = User.getCurrent();
143 * @param c
144 * @param currentfs We return this if we did not make a new one.
145 * @param uniqueName Same name used to help identify the created fs.
146 * @return A new fs instance if we are up on DistributeFileSystem.
147 * @throws IOException
150 @Override
151 protected void handleReportForDutyResponse(
152 final RegionServerStartupResponse c) throws IOException {
153 super.handleReportForDutyResponse(c);
154 // Run this thread to shutdown our filesystem on way out.
155 this.shutdownThread = new SingleFileSystemShutdownThread(getFileSystem());
158 @Override
159 public void run() {
160 try {
161 this.user.runAs(new PrivilegedAction<Object>() {
162 @Override
163 public Object run() {
164 runRegionServer();
165 return null;
168 } catch (Throwable t) {
169 LOG.error("Exception in run", t);
170 } finally {
171 // Run this on the way out.
172 if (this.shutdownThread != null) {
173 this.shutdownThread.start();
174 Threads.shutdown(this.shutdownThread, 30000);
179 private void runRegionServer() {
180 super.run();
183 @Override
184 protected void kill() {
185 killedServers.add(getServerName());
186 super.kill();
189 @Override
190 public void abort(final String reason, final Throwable cause) {
191 this.user.runAs(new PrivilegedAction<Object>() {
192 @Override
193 public Object run() {
194 abortRegionServer(reason, cause);
195 return null;
200 private void abortRegionServer(String reason, Throwable cause) {
201 super.abort(reason, cause);
206 * Alternate shutdown hook.
207 * Just shuts down the passed fs, not all as default filesystem hook does.
209 static class SingleFileSystemShutdownThread extends Thread {
210 private final FileSystem fs;
211 SingleFileSystemShutdownThread(final FileSystem fs) {
212 super("Shutdown of " + fs);
213 this.fs = fs;
215 @Override
216 public void run() {
217 try {
218 LOG.info("Hook closing fs=" + this.fs);
219 this.fs.close();
220 } catch (NullPointerException npe) {
221 LOG.debug("Need to fix these: " + npe.toString());
222 } catch (IOException e) {
223 LOG.warn("Running hook", e);
228 private void init(final int nMasterNodes, final int numAlwaysStandByMasters,
229 final int nRegionNodes, List<Integer> rsPorts, Class<? extends HMaster> masterClass,
230 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> regionserverClass)
231 throws IOException, InterruptedException {
232 try {
233 if (masterClass == null){
234 masterClass = HMaster.class;
236 if (regionserverClass == null){
237 regionserverClass = MiniHBaseCluster.MiniHBaseClusterRegionServer.class;
240 // start up a LocalHBaseCluster
241 hbaseCluster = new LocalHBaseCluster(conf, nMasterNodes, numAlwaysStandByMasters, 0,
242 masterClass, regionserverClass);
244 // manually add the regionservers as other users
245 for (int i = 0; i < nRegionNodes; i++) {
246 Configuration rsConf = HBaseConfiguration.create(conf);
247 if (rsPorts != null) {
248 rsConf.setInt(HConstants.REGIONSERVER_PORT, rsPorts.get(i));
250 User user = HBaseTestingUtility.getDifferentUser(rsConf,
251 ".hfs."+index++);
252 hbaseCluster.addRegionServer(rsConf, i, user);
255 hbaseCluster.startup();
256 } catch (IOException e) {
257 shutdown();
258 throw e;
259 } catch (Throwable t) {
260 LOG.error("Error starting cluster", t);
261 shutdown();
262 throw new IOException("Shutting down", t);
266 @Override
267 public void startRegionServer(String hostname, int port) throws IOException {
268 final Configuration newConf = HBaseConfiguration.create(conf);
269 newConf.setInt(HConstants.REGIONSERVER_PORT, port);
270 startRegionServer(newConf);
273 @Override
274 public void killRegionServer(ServerName serverName) throws IOException {
275 HRegionServer server = getRegionServer(getRegionServerIndex(serverName));
276 if (server instanceof MiniHBaseClusterRegionServer) {
277 LOG.info("Killing " + server.toString());
278 ((MiniHBaseClusterRegionServer) server).kill();
279 } else {
280 abortRegionServer(getRegionServerIndex(serverName));
284 @Override
285 public boolean isKilledRS(ServerName serverName) {
286 return MiniHBaseClusterRegionServer.killedServers.contains(serverName);
289 @Override
290 public void stopRegionServer(ServerName serverName) throws IOException {
291 stopRegionServer(getRegionServerIndex(serverName));
294 @Override
295 public void suspendRegionServer(ServerName serverName) throws IOException {
296 suspendRegionServer(getRegionServerIndex(serverName));
299 @Override
300 public void resumeRegionServer(ServerName serverName) throws IOException {
301 resumeRegionServer(getRegionServerIndex(serverName));
304 @Override
305 public void waitForRegionServerToStop(ServerName serverName, long timeout) throws IOException {
306 //ignore timeout for now
307 waitOnRegionServer(getRegionServerIndex(serverName));
310 @Override
311 public void startZkNode(String hostname, int port) throws IOException {
312 LOG.warn("Starting zookeeper nodes on mini cluster is not supported");
315 @Override
316 public void killZkNode(ServerName serverName) throws IOException {
317 LOG.warn("Aborting zookeeper nodes on mini cluster is not supported");
320 @Override
321 public void stopZkNode(ServerName serverName) throws IOException {
322 LOG.warn("Stopping zookeeper nodes on mini cluster is not supported");
325 @Override
326 public void waitForZkNodeToStart(ServerName serverName, long timeout) throws IOException {
327 LOG.warn("Waiting for zookeeper nodes to start on mini cluster is not supported");
330 @Override
331 public void waitForZkNodeToStop(ServerName serverName, long timeout) throws IOException {
332 LOG.warn("Waiting for zookeeper nodes to stop on mini cluster is not supported");
335 @Override
336 public void startDataNode(ServerName serverName) throws IOException {
337 LOG.warn("Starting datanodes on mini cluster is not supported");
340 @Override
341 public void killDataNode(ServerName serverName) throws IOException {
342 LOG.warn("Aborting datanodes on mini cluster is not supported");
345 @Override
346 public void stopDataNode(ServerName serverName) throws IOException {
347 LOG.warn("Stopping datanodes on mini cluster is not supported");
350 @Override
351 public void waitForDataNodeToStart(ServerName serverName, long timeout) throws IOException {
352 LOG.warn("Waiting for datanodes to start on mini cluster is not supported");
355 @Override
356 public void waitForDataNodeToStop(ServerName serverName, long timeout) throws IOException {
357 LOG.warn("Waiting for datanodes to stop on mini cluster is not supported");
360 @Override
361 public void startNameNode(ServerName serverName) throws IOException {
362 LOG.warn("Starting namenodes on mini cluster is not supported");
365 @Override
366 public void killNameNode(ServerName serverName) throws IOException {
367 LOG.warn("Aborting namenodes on mini cluster is not supported");
370 @Override
371 public void stopNameNode(ServerName serverName) throws IOException {
372 LOG.warn("Stopping namenodes on mini cluster is not supported");
375 @Override
376 public void waitForNameNodeToStart(ServerName serverName, long timeout) throws IOException {
377 LOG.warn("Waiting for namenodes to start on mini cluster is not supported");
380 @Override
381 public void waitForNameNodeToStop(ServerName serverName, long timeout) throws IOException {
382 LOG.warn("Waiting for namenodes to stop on mini cluster is not supported");
385 @Override
386 public void startMaster(String hostname, int port) throws IOException {
387 this.startMaster();
390 @Override
391 public void killMaster(ServerName serverName) throws IOException {
392 abortMaster(getMasterIndex(serverName));
395 @Override
396 public void stopMaster(ServerName serverName) throws IOException {
397 stopMaster(getMasterIndex(serverName));
400 @Override
401 public void waitForMasterToStop(ServerName serverName, long timeout) throws IOException {
402 //ignore timeout for now
403 waitOnMaster(getMasterIndex(serverName));
407 * Starts a region server thread running
409 * @throws IOException
410 * @return New RegionServerThread
412 public JVMClusterUtil.RegionServerThread startRegionServer()
413 throws IOException {
414 final Configuration newConf = HBaseConfiguration.create(conf);
415 return startRegionServer(newConf);
418 private JVMClusterUtil.RegionServerThread startRegionServer(Configuration configuration)
419 throws IOException {
420 User rsUser =
421 HBaseTestingUtility.getDifferentUser(configuration, ".hfs."+index++);
422 JVMClusterUtil.RegionServerThread t = null;
423 try {
424 t = hbaseCluster.addRegionServer(
425 configuration, hbaseCluster.getRegionServers().size(), rsUser);
426 t.start();
427 t.waitForServerOnline();
428 } catch (InterruptedException ie) {
429 throw new IOException("Interrupted adding regionserver to cluster", ie);
431 return t;
435 * Starts a region server thread and waits until its processed by master. Throws an exception
436 * when it can't start a region server or when the region server is not processed by master
437 * within the timeout.
439 * @return New RegionServerThread
441 public JVMClusterUtil.RegionServerThread startRegionServerAndWait(long timeout)
442 throws IOException {
444 JVMClusterUtil.RegionServerThread t = startRegionServer();
445 ServerName rsServerName = t.getRegionServer().getServerName();
447 long start = System.currentTimeMillis();
448 ClusterMetrics clusterStatus = getClusterMetrics();
449 while ((System.currentTimeMillis() - start) < timeout) {
450 if (clusterStatus != null && clusterStatus.getLiveServerMetrics().containsKey(rsServerName)) {
451 return t;
453 Threads.sleep(100);
455 if (t.getRegionServer().isOnline()) {
456 throw new IOException("RS: " + rsServerName + " online, but not processed by master");
457 } else {
458 throw new IOException("RS: " + rsServerName + " is offline");
463 * Cause a region server to exit doing basic clean up only on its way out.
464 * @param serverNumber Used as index into a list.
466 public String abortRegionServer(int serverNumber) {
467 HRegionServer server = getRegionServer(serverNumber);
468 LOG.info("Aborting " + server.toString());
469 server.abort("Aborting for tests", new Exception("Trace info"));
470 return server.toString();
474 * Shut down the specified region server cleanly
476 * @param serverNumber Used as index into a list.
477 * @return the region server that was stopped
479 public JVMClusterUtil.RegionServerThread stopRegionServer(int serverNumber) {
480 return stopRegionServer(serverNumber, true);
484 * Shut down the specified region server cleanly
486 * @param serverNumber Used as index into a list.
487 * @param shutdownFS True is we are to shutdown the filesystem as part of this
488 * regionserver's shutdown. Usually we do but you do not want to do this if
489 * you are running multiple regionservers in a test and you shut down one
490 * before end of the test.
491 * @return the region server that was stopped
493 public JVMClusterUtil.RegionServerThread stopRegionServer(int serverNumber,
494 final boolean shutdownFS) {
495 JVMClusterUtil.RegionServerThread server =
496 hbaseCluster.getRegionServers().get(serverNumber);
497 LOG.info("Stopping " + server.toString());
498 server.getRegionServer().stop("Stopping rs " + serverNumber);
499 return server;
503 * Suspend the specified region server
504 * @param serverNumber Used as index into a list.
505 * @return
507 public JVMClusterUtil.RegionServerThread suspendRegionServer(int serverNumber) {
508 JVMClusterUtil.RegionServerThread server =
509 hbaseCluster.getRegionServers().get(serverNumber);
510 LOG.info("Suspending {}", server.toString());
511 server.suspend();
512 return server;
516 * Resume the specified region server
517 * @param serverNumber Used as index into a list.
518 * @return
520 public JVMClusterUtil.RegionServerThread resumeRegionServer(int serverNumber) {
521 JVMClusterUtil.RegionServerThread server =
522 hbaseCluster.getRegionServers().get(serverNumber);
523 LOG.info("Resuming {}", server.toString());
524 server.resume();
525 return server;
529 * Wait for the specified region server to stop. Removes this thread from list
530 * of running threads.
531 * @param serverNumber
532 * @return Name of region server that just went down.
534 public String waitOnRegionServer(final int serverNumber) {
535 return this.hbaseCluster.waitOnRegionServer(serverNumber);
540 * Starts a master thread running
542 * @return New RegionServerThread
544 public JVMClusterUtil.MasterThread startMaster() throws IOException {
545 Configuration c = HBaseConfiguration.create(conf);
546 User user =
547 HBaseTestingUtility.getDifferentUser(c, ".hfs."+index++);
549 JVMClusterUtil.MasterThread t = null;
550 try {
551 t = hbaseCluster.addMaster(c, hbaseCluster.getMasters().size(), user);
552 t.start();
553 } catch (InterruptedException ie) {
554 throw new IOException("Interrupted adding master to cluster", ie);
556 conf.set(HConstants.MASTER_ADDRS_KEY,
557 hbaseCluster.getConfiguration().get(HConstants.MASTER_ADDRS_KEY));
558 return t;
562 * Returns the current active master, if available.
563 * @return the active HMaster, null if none is active.
565 public HMaster getMaster() {
566 return this.hbaseCluster.getActiveMaster();
570 * Returns the current active master thread, if available.
571 * @return the active MasterThread, null if none is active.
573 public MasterThread getMasterThread() {
574 for (MasterThread mt: hbaseCluster.getLiveMasters()) {
575 if (mt.getMaster().isActiveMaster()) {
576 return mt;
579 return null;
583 * Returns the master at the specified index, if available.
584 * @return the active HMaster, null if none is active.
586 public HMaster getMaster(final int serverNumber) {
587 return this.hbaseCluster.getMaster(serverNumber);
591 * Cause a master to exit without shutting down entire cluster.
592 * @param serverNumber Used as index into a list.
594 public String abortMaster(int serverNumber) {
595 HMaster server = getMaster(serverNumber);
596 LOG.info("Aborting " + server.toString());
597 server.abort("Aborting for tests", new Exception("Trace info"));
598 return server.toString();
602 * Shut down the specified master cleanly
604 * @param serverNumber Used as index into a list.
605 * @return the region server that was stopped
607 public JVMClusterUtil.MasterThread stopMaster(int serverNumber) {
608 return stopMaster(serverNumber, true);
612 * Shut down the specified master cleanly
614 * @param serverNumber Used as index into a list.
615 * @param shutdownFS True is we are to shutdown the filesystem as part of this
616 * master's shutdown. Usually we do but you do not want to do this if
617 * you are running multiple master in a test and you shut down one
618 * before end of the test.
619 * @return the master that was stopped
621 public JVMClusterUtil.MasterThread stopMaster(int serverNumber,
622 final boolean shutdownFS) {
623 JVMClusterUtil.MasterThread server =
624 hbaseCluster.getMasters().get(serverNumber);
625 LOG.info("Stopping " + server.toString());
626 server.getMaster().stop("Stopping master " + serverNumber);
627 return server;
631 * Wait for the specified master to stop. Removes this thread from list
632 * of running threads.
633 * @param serverNumber
634 * @return Name of master that just went down.
636 public String waitOnMaster(final int serverNumber) {
637 return this.hbaseCluster.waitOnMaster(serverNumber);
641 * Blocks until there is an active master and that master has completed
642 * initialization.
644 * @return true if an active master becomes available. false if there are no
645 * masters left.
646 * @throws InterruptedException
648 @Override
649 public boolean waitForActiveAndReadyMaster(long timeout) throws IOException {
650 List<JVMClusterUtil.MasterThread> mts;
651 long start = System.currentTimeMillis();
652 while (!(mts = getMasterThreads()).isEmpty()
653 && (System.currentTimeMillis() - start) < timeout) {
654 for (JVMClusterUtil.MasterThread mt : mts) {
655 if (mt.getMaster().isActiveMaster() && mt.getMaster().isInitialized()) {
656 return true;
660 Threads.sleep(100);
662 return false;
666 * @return List of master threads.
668 public List<JVMClusterUtil.MasterThread> getMasterThreads() {
669 return this.hbaseCluster.getMasters();
673 * @return List of live master threads (skips the aborted and the killed)
675 public List<JVMClusterUtil.MasterThread> getLiveMasterThreads() {
676 return this.hbaseCluster.getLiveMasters();
680 * Wait for Mini HBase Cluster to shut down.
682 public void join() {
683 this.hbaseCluster.join();
687 * Shut down the mini HBase cluster
689 @Override
690 public void shutdown() throws IOException {
691 if (this.hbaseCluster != null) {
692 this.hbaseCluster.shutdown();
696 @Override
697 public void close() throws IOException {
700 @Override
701 public ClusterMetrics getClusterMetrics() throws IOException {
702 HMaster master = getMaster();
703 return master == null ? null : master.getClusterMetrics();
706 private void executeFlush(HRegion region) throws IOException {
707 // retry 5 times if we can not flush
708 for (int i = 0; i < 5; i++) {
709 FlushResult result = region.flush(true);
710 if (result.getResult() != FlushResult.Result.CANNOT_FLUSH) {
711 return;
713 Threads.sleep(1000);
718 * Call flushCache on all regions on all participating regionservers.
720 public void flushcache() throws IOException {
721 for (JVMClusterUtil.RegionServerThread t : this.hbaseCluster.getRegionServers()) {
722 for (HRegion r : t.getRegionServer().getOnlineRegionsLocalContext()) {
723 executeFlush(r);
729 * Call flushCache on all regions of the specified table.
731 public void flushcache(TableName tableName) throws IOException {
732 for (JVMClusterUtil.RegionServerThread t : this.hbaseCluster.getRegionServers()) {
733 for (HRegion r : t.getRegionServer().getOnlineRegionsLocalContext()) {
734 if (r.getTableDescriptor().getTableName().equals(tableName)) {
735 executeFlush(r);
742 * Call flushCache on all regions on all participating regionservers.
743 * @throws IOException
745 public void compact(boolean major) throws IOException {
746 for (JVMClusterUtil.RegionServerThread t:
747 this.hbaseCluster.getRegionServers()) {
748 for(HRegion r: t.getRegionServer().getOnlineRegionsLocalContext()) {
749 r.compact(major);
755 * Call flushCache on all regions of the specified table.
756 * @throws IOException
758 public void compact(TableName tableName, boolean major) throws IOException {
759 for (JVMClusterUtil.RegionServerThread t:
760 this.hbaseCluster.getRegionServers()) {
761 for(HRegion r: t.getRegionServer().getOnlineRegionsLocalContext()) {
762 if(r.getTableDescriptor().getTableName().equals(tableName)) {
763 r.compact(major);
770 * @return Number of live region servers in the cluster currently.
772 public int getNumLiveRegionServers() {
773 return this.hbaseCluster.getLiveRegionServers().size();
777 * @return List of region server threads. Does not return the master even though it is also
778 * a region server.
780 public List<JVMClusterUtil.RegionServerThread> getRegionServerThreads() {
781 return this.hbaseCluster.getRegionServers();
785 * @return List of live region server threads (skips the aborted and the killed)
787 public List<JVMClusterUtil.RegionServerThread> getLiveRegionServerThreads() {
788 return this.hbaseCluster.getLiveRegionServers();
792 * Grab a numbered region server of your choice.
793 * @param serverNumber
794 * @return region server
796 public HRegionServer getRegionServer(int serverNumber) {
797 return hbaseCluster.getRegionServer(serverNumber);
800 public HRegionServer getRegionServer(ServerName serverName) {
801 return hbaseCluster.getRegionServers().stream()
802 .map(t -> t.getRegionServer())
803 .filter(r -> r.getServerName().equals(serverName))
804 .findFirst().orElse(null);
807 public List<HRegion> getRegions(byte[] tableName) {
808 return getRegions(TableName.valueOf(tableName));
811 public List<HRegion> getRegions(TableName tableName) {
812 List<HRegion> ret = new ArrayList<>();
813 for (JVMClusterUtil.RegionServerThread rst : getRegionServerThreads()) {
814 HRegionServer hrs = rst.getRegionServer();
815 for (Region region : hrs.getOnlineRegionsLocalContext()) {
816 if (region.getTableDescriptor().getTableName().equals(tableName)) {
817 ret.add((HRegion)region);
821 return ret;
825 * @return Index into List of {@link MiniHBaseCluster#getRegionServerThreads()}
826 * of HRS carrying regionName. Returns -1 if none found.
828 public int getServerWithMeta() {
829 return getServerWith(HRegionInfo.FIRST_META_REGIONINFO.getRegionName());
833 * Get the location of the specified region
834 * @param regionName Name of the region in bytes
835 * @return Index into List of {@link MiniHBaseCluster#getRegionServerThreads()}
836 * of HRS carrying hbase:meta. Returns -1 if none found.
838 public int getServerWith(byte[] regionName) {
839 int index = -1;
840 int count = 0;
841 for (JVMClusterUtil.RegionServerThread rst: getRegionServerThreads()) {
842 HRegionServer hrs = rst.getRegionServer();
843 if (!hrs.isStopped()) {
844 Region region = hrs.getOnlineRegion(regionName);
845 if (region != null) {
846 index = count;
847 break;
850 count++;
852 return index;
855 @Override
856 public ServerName getServerHoldingRegion(final TableName tn, byte[] regionName)
857 throws IOException {
858 // Assume there is only one master thread which is the active master.
859 // If there are multiple master threads, the backup master threads
860 // should hold some regions. Please refer to #countServedRegions
861 // to see how we find out all regions.
862 HMaster master = getMaster();
863 Region region = master.getOnlineRegion(regionName);
864 if (region != null) {
865 return master.getServerName();
867 int index = getServerWith(regionName);
868 if (index < 0) {
869 return null;
871 return getRegionServer(index).getServerName();
875 * Counts the total numbers of regions being served by the currently online
876 * region servers by asking each how many regions they have. Does not look
877 * at hbase:meta at all. Count includes catalog tables.
878 * @return number of regions being served by all region servers
880 public long countServedRegions() {
881 long count = 0;
882 for (JVMClusterUtil.RegionServerThread rst : getLiveRegionServerThreads()) {
883 count += rst.getRegionServer().getNumberOfOnlineRegions();
885 for (JVMClusterUtil.MasterThread mt : getLiveMasterThreads()) {
886 count += mt.getMaster().getNumberOfOnlineRegions();
888 return count;
892 * Do a simulated kill all masters and regionservers. Useful when it is
893 * impossible to bring the mini-cluster back for clean shutdown.
895 public void killAll() {
896 // Do backups first.
897 MasterThread activeMaster = null;
898 for (MasterThread masterThread : getMasterThreads()) {
899 if (!masterThread.getMaster().isActiveMaster()) {
900 masterThread.getMaster().abort("killAll");
901 } else {
902 activeMaster = masterThread;
905 // Do active after.
906 if (activeMaster != null) {
907 activeMaster.getMaster().abort("killAll");
909 for (RegionServerThread rst : getRegionServerThreads()) {
910 rst.getRegionServer().abort("killAll");
914 @Override
915 public void waitUntilShutDown() {
916 this.hbaseCluster.join();
919 public List<HRegion> findRegionsForTable(TableName tableName) {
920 ArrayList<HRegion> ret = new ArrayList<>();
921 for (JVMClusterUtil.RegionServerThread rst : getRegionServerThreads()) {
922 HRegionServer hrs = rst.getRegionServer();
923 for (Region region : hrs.getRegions(tableName)) {
924 if (region.getTableDescriptor().getTableName().equals(tableName)) {
925 ret.add((HRegion)region);
929 return ret;
933 protected int getRegionServerIndex(ServerName serverName) {
934 //we have a small number of region servers, this should be fine for now.
935 List<RegionServerThread> servers = getRegionServerThreads();
936 for (int i=0; i < servers.size(); i++) {
937 if (servers.get(i).getRegionServer().getServerName().equals(serverName)) {
938 return i;
941 return -1;
944 protected int getMasterIndex(ServerName serverName) {
945 List<MasterThread> masters = getMasterThreads();
946 for (int i = 0; i < masters.size(); i++) {
947 if (masters.get(i).getMaster().getServerName().equals(serverName)) {
948 return i;
951 return -1;