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
;
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
;
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
;
59 * Start a MiniHBaseCluster.
60 * @param conf Configuration to be used for cluster
61 * @param numRegionServers initial number of region servers to start.
64 public MiniHBaseCluster(Configuration conf
, int numRegionServers
)
65 throws IOException
, InterruptedException
{
66 this(conf
, 1, numRegionServers
);
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.
76 public MiniHBaseCluster(Configuration conf
, int numMasters
, int numRegionServers
)
77 throws IOException
, InterruptedException
{
78 this(conf
, numMasters
, numRegionServers
, null, null);
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
);
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.
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
{
109 CompatibilityFactory
.getInstance(MetricsAssertHelper
.class).init();
111 init(numMasters
, numAlwaysStandByMasters
, numRegionServers
, rsPorts
, masterClass
,
113 this.initialClusterStatus
= getClusterMetrics();
116 public Configuration
getConfiguration() {
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
{
139 this.user
= User
.getCurrent();
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
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());
161 this.user
.runAs(new PrivilegedAction
<Object
>() {
163 public Object
run() {
168 } catch (Throwable t
) {
169 LOG
.error("Exception in run", t
);
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() {
184 protected void kill() {
185 killedServers
.add(getServerName());
190 public void abort(final String reason
, final Throwable cause
) {
191 this.user
.runAs(new PrivilegedAction
<Object
>() {
193 public Object
run() {
194 abortRegionServer(reason
, cause
);
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
);
218 LOG
.info("Hook closing fs=" + this.fs
);
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
{
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
,
252 hbaseCluster
.addRegionServer(rsConf
, i
, user
);
255 hbaseCluster
.startup();
256 } catch (IOException e
) {
259 } catch (Throwable t
) {
260 LOG
.error("Error starting cluster", t
);
262 throw new IOException("Shutting down", t
);
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
);
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();
280 abortRegionServer(getRegionServerIndex(serverName
));
285 public boolean isKilledRS(ServerName serverName
) {
286 return MiniHBaseClusterRegionServer
.killedServers
.contains(serverName
);
290 public void stopRegionServer(ServerName serverName
) throws IOException
{
291 stopRegionServer(getRegionServerIndex(serverName
));
295 public void suspendRegionServer(ServerName serverName
) throws IOException
{
296 suspendRegionServer(getRegionServerIndex(serverName
));
300 public void resumeRegionServer(ServerName serverName
) throws IOException
{
301 resumeRegionServer(getRegionServerIndex(serverName
));
305 public void waitForRegionServerToStop(ServerName serverName
, long timeout
) throws IOException
{
306 //ignore timeout for now
307 waitOnRegionServer(getRegionServerIndex(serverName
));
311 public void startZkNode(String hostname
, int port
) throws IOException
{
312 LOG
.warn("Starting zookeeper nodes on mini cluster is not supported");
316 public void killZkNode(ServerName serverName
) throws IOException
{
317 LOG
.warn("Aborting zookeeper nodes on mini cluster is not supported");
321 public void stopZkNode(ServerName serverName
) throws IOException
{
322 LOG
.warn("Stopping zookeeper nodes on mini cluster is not supported");
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");
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");
336 public void startDataNode(ServerName serverName
) throws IOException
{
337 LOG
.warn("Starting datanodes on mini cluster is not supported");
341 public void killDataNode(ServerName serverName
) throws IOException
{
342 LOG
.warn("Aborting datanodes on mini cluster is not supported");
346 public void stopDataNode(ServerName serverName
) throws IOException
{
347 LOG
.warn("Stopping datanodes on mini cluster is not supported");
351 public void waitForDataNodeToStart(ServerName serverName
, long timeout
) throws IOException
{
352 LOG
.warn("Waiting for datanodes to start on mini cluster is not supported");
356 public void waitForDataNodeToStop(ServerName serverName
, long timeout
) throws IOException
{
357 LOG
.warn("Waiting for datanodes to stop on mini cluster is not supported");
361 public void startNameNode(ServerName serverName
) throws IOException
{
362 LOG
.warn("Starting namenodes on mini cluster is not supported");
366 public void killNameNode(ServerName serverName
) throws IOException
{
367 LOG
.warn("Aborting namenodes on mini cluster is not supported");
371 public void stopNameNode(ServerName serverName
) throws IOException
{
372 LOG
.warn("Stopping namenodes on mini cluster is not supported");
376 public void waitForNameNodeToStart(ServerName serverName
, long timeout
) throws IOException
{
377 LOG
.warn("Waiting for namenodes to start on mini cluster is not supported");
381 public void waitForNameNodeToStop(ServerName serverName
, long timeout
) throws IOException
{
382 LOG
.warn("Waiting for namenodes to stop on mini cluster is not supported");
386 public void startMaster(String hostname
, int port
) throws IOException
{
391 public void killMaster(ServerName serverName
) throws IOException
{
392 abortMaster(getMasterIndex(serverName
));
396 public void stopMaster(ServerName serverName
) throws IOException
{
397 stopMaster(getMasterIndex(serverName
));
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()
414 final Configuration newConf
= HBaseConfiguration
.create(conf
);
415 return startRegionServer(newConf
);
418 private JVMClusterUtil
.RegionServerThread
startRegionServer(Configuration configuration
)
421 HBaseTestingUtility
.getDifferentUser(configuration
, ".hfs."+index
++);
422 JVMClusterUtil
.RegionServerThread t
= null;
424 t
= hbaseCluster
.addRegionServer(
425 configuration
, hbaseCluster
.getRegionServers().size(), rsUser
);
427 t
.waitForServerOnline();
428 } catch (InterruptedException ie
) {
429 throw new IOException("Interrupted adding regionserver to cluster", ie
);
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
)
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
)) {
455 if (t
.getRegionServer().isOnline()) {
456 throw new IOException("RS: " + rsServerName
+ " online, but not processed by master");
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
);
503 * Suspend the specified region server
504 * @param serverNumber Used as index into a list.
507 public JVMClusterUtil
.RegionServerThread
suspendRegionServer(int serverNumber
) {
508 JVMClusterUtil
.RegionServerThread server
=
509 hbaseCluster
.getRegionServers().get(serverNumber
);
510 LOG
.info("Suspending {}", server
.toString());
516 * Resume the specified region server
517 * @param serverNumber Used as index into a list.
520 public JVMClusterUtil
.RegionServerThread
resumeRegionServer(int serverNumber
) {
521 JVMClusterUtil
.RegionServerThread server
=
522 hbaseCluster
.getRegionServers().get(serverNumber
);
523 LOG
.info("Resuming {}", server
.toString());
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
);
547 HBaseTestingUtility
.getDifferentUser(c
, ".hfs."+index
++);
549 JVMClusterUtil
.MasterThread t
= null;
551 t
= hbaseCluster
.addMaster(c
, hbaseCluster
.getMasters().size(), user
);
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
));
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()) {
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
);
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
644 * @return true if an active master becomes available. false if there are no
646 * @throws InterruptedException
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()) {
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.
683 this.hbaseCluster
.join();
687 * Shut down the mini HBase cluster
690 public void shutdown() throws IOException
{
691 if (this.hbaseCluster
!= null) {
692 this.hbaseCluster
.shutdown();
697 public void close() throws IOException
{
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
) {
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()) {
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
)) {
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()) {
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
)) {
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
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
);
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
) {
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) {
856 public ServerName
getServerHoldingRegion(final TableName tn
, byte[] regionName
)
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
);
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() {
882 for (JVMClusterUtil
.RegionServerThread rst
: getLiveRegionServerThreads()) {
883 count
+= rst
.getRegionServer().getNumberOfOnlineRegions();
885 for (JVMClusterUtil
.MasterThread mt
: getLiveMasterThreads()) {
886 count
+= mt
.getMaster().getNumberOfOnlineRegions();
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() {
897 MasterThread activeMaster
= null;
898 for (MasterThread masterThread
: getMasterThreads()) {
899 if (!masterThread
.getMaster().isActiveMaster()) {
900 masterThread
.getMaster().abort("killAll");
902 activeMaster
= masterThread
;
906 if (activeMaster
!= null) {
907 activeMaster
.getMaster().abort("killAll");
909 for (RegionServerThread rst
: getRegionServerThreads()) {
910 rst
.getRegionServer().abort("killAll");
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
);
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
)) {
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
)) {