HBASE-23055 Alter hbase:meta (#667)
[hbase.git] / hbase-server / src / test / java / org / apache / hadoop / hbase / HBaseTestingUtility.java
blob54c5c17bf675bd25c0222c01e10986a362c97d46
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase;
20 import static org.junit.Assert.assertEquals;
21 import static org.junit.Assert.assertTrue;
22 import static org.junit.Assert.fail;
24 import edu.umd.cs.findbugs.annotations.Nullable;
25 import java.io.File;
26 import java.io.IOException;
27 import java.io.OutputStream;
28 import java.lang.reflect.Field;
29 import java.lang.reflect.Modifier;
30 import java.net.BindException;
31 import java.net.DatagramSocket;
32 import java.net.InetAddress;
33 import java.net.ServerSocket;
34 import java.net.Socket;
35 import java.net.UnknownHostException;
36 import java.nio.charset.StandardCharsets;
37 import java.security.MessageDigest;
38 import java.util.ArrayList;
39 import java.util.Arrays;
40 import java.util.Collection;
41 import java.util.Collections;
42 import java.util.EnumSet;
43 import java.util.HashSet;
44 import java.util.Iterator;
45 import java.util.List;
46 import java.util.Map;
47 import java.util.NavigableSet;
48 import java.util.Properties;
49 import java.util.Random;
50 import java.util.Set;
51 import java.util.TreeSet;
52 import java.util.concurrent.TimeUnit;
53 import java.util.concurrent.atomic.AtomicReference;
54 import org.apache.commons.io.FileUtils;
55 import org.apache.commons.lang3.RandomStringUtils;
56 import org.apache.commons.logging.impl.Jdk14Logger;
57 import org.apache.commons.logging.impl.Log4JLogger;
58 import org.apache.hadoop.conf.Configuration;
59 import org.apache.hadoop.fs.FileSystem;
60 import org.apache.hadoop.fs.Path;
61 import org.apache.hadoop.hbase.ClusterMetrics.Option;
62 import org.apache.hadoop.hbase.Waiter.ExplainingPredicate;
63 import org.apache.hadoop.hbase.Waiter.Predicate;
64 import org.apache.hadoop.hbase.client.Admin;
65 import org.apache.hadoop.hbase.client.AsyncClusterConnection;
66 import org.apache.hadoop.hbase.client.BufferedMutator;
67 import org.apache.hadoop.hbase.client.ClusterConnectionFactory;
68 import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
69 import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
70 import org.apache.hadoop.hbase.client.Connection;
71 import org.apache.hadoop.hbase.client.ConnectionFactory;
72 import org.apache.hadoop.hbase.client.Consistency;
73 import org.apache.hadoop.hbase.client.Delete;
74 import org.apache.hadoop.hbase.client.Durability;
75 import org.apache.hadoop.hbase.client.Get;
76 import org.apache.hadoop.hbase.client.Hbck;
77 import org.apache.hadoop.hbase.client.Put;
78 import org.apache.hadoop.hbase.client.RegionInfo;
79 import org.apache.hadoop.hbase.client.RegionInfoBuilder;
80 import org.apache.hadoop.hbase.client.RegionLocator;
81 import org.apache.hadoop.hbase.client.Result;
82 import org.apache.hadoop.hbase.client.ResultScanner;
83 import org.apache.hadoop.hbase.client.Scan;
84 import org.apache.hadoop.hbase.client.Table;
85 import org.apache.hadoop.hbase.client.TableDescriptor;
86 import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
87 import org.apache.hadoop.hbase.client.TableState;
88 import org.apache.hadoop.hbase.fs.HFileSystem;
89 import org.apache.hadoop.hbase.io.compress.Compression;
90 import org.apache.hadoop.hbase.io.compress.Compression.Algorithm;
91 import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
92 import org.apache.hadoop.hbase.io.hfile.BlockCache;
93 import org.apache.hadoop.hbase.io.hfile.ChecksumUtil;
94 import org.apache.hadoop.hbase.io.hfile.HFile;
95 import org.apache.hadoop.hbase.ipc.RpcServerInterface;
96 import org.apache.hadoop.hbase.ipc.ServerNotRunningYetException;
97 import org.apache.hadoop.hbase.mapreduce.MapreduceTestingShim;
98 import org.apache.hadoop.hbase.master.HMaster;
99 import org.apache.hadoop.hbase.master.RegionState;
100 import org.apache.hadoop.hbase.master.ServerManager;
101 import org.apache.hadoop.hbase.master.assignment.AssignmentManager;
102 import org.apache.hadoop.hbase.master.assignment.AssignmentTestingUtil;
103 import org.apache.hadoop.hbase.master.assignment.RegionStateStore;
104 import org.apache.hadoop.hbase.master.assignment.RegionStates;
105 import org.apache.hadoop.hbase.mob.MobFileCache;
106 import org.apache.hadoop.hbase.regionserver.BloomType;
107 import org.apache.hadoop.hbase.regionserver.ChunkCreator;
108 import org.apache.hadoop.hbase.regionserver.HRegion;
109 import org.apache.hadoop.hbase.regionserver.HRegionServer;
110 import org.apache.hadoop.hbase.regionserver.HStore;
111 import org.apache.hadoop.hbase.regionserver.InternalScanner;
112 import org.apache.hadoop.hbase.regionserver.MemStoreLABImpl;
113 import org.apache.hadoop.hbase.regionserver.Region;
114 import org.apache.hadoop.hbase.regionserver.RegionScanner;
115 import org.apache.hadoop.hbase.regionserver.RegionServerServices;
116 import org.apache.hadoop.hbase.regionserver.RegionServerStoppedException;
117 import org.apache.hadoop.hbase.security.HBaseKerberosUtils;
118 import org.apache.hadoop.hbase.security.User;
119 import org.apache.hadoop.hbase.security.UserProvider;
120 import org.apache.hadoop.hbase.security.visibility.VisibilityLabelsCache;
121 import org.apache.hadoop.hbase.trace.TraceUtil;
122 import org.apache.hadoop.hbase.util.Bytes;
123 import org.apache.hadoop.hbase.util.CommonFSUtils;
124 import org.apache.hadoop.hbase.util.FSUtils;
125 import org.apache.hadoop.hbase.util.JVMClusterUtil;
126 import org.apache.hadoop.hbase.util.JVMClusterUtil.MasterThread;
127 import org.apache.hadoop.hbase.util.JVMClusterUtil.RegionServerThread;
128 import org.apache.hadoop.hbase.util.Pair;
129 import org.apache.hadoop.hbase.util.RegionSplitter;
130 import org.apache.hadoop.hbase.util.RegionSplitter.SplitAlgorithm;
131 import org.apache.hadoop.hbase.util.RetryCounter;
132 import org.apache.hadoop.hbase.util.Threads;
133 import org.apache.hadoop.hbase.wal.WAL;
134 import org.apache.hadoop.hbase.wal.WALFactory;
135 import org.apache.hadoop.hbase.zookeeper.EmptyWatcher;
136 import org.apache.hadoop.hbase.zookeeper.ZKConfig;
137 import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
138 import org.apache.hadoop.hdfs.DFSClient;
139 import org.apache.hadoop.hdfs.DistributedFileSystem;
140 import org.apache.hadoop.hdfs.MiniDFSCluster;
141 import org.apache.hadoop.hdfs.server.namenode.EditLogFileOutputStream;
142 import org.apache.hadoop.mapred.JobConf;
143 import org.apache.hadoop.mapred.MiniMRCluster;
144 import org.apache.hadoop.mapred.TaskLog;
145 import org.apache.hadoop.minikdc.MiniKdc;
146 import org.apache.log4j.LogManager;
147 import org.apache.yetus.audience.InterfaceAudience;
148 import org.apache.zookeeper.WatchedEvent;
149 import org.apache.zookeeper.ZooKeeper;
150 import org.apache.zookeeper.ZooKeeper.States;
151 import org.slf4j.Logger;
152 import org.slf4j.LoggerFactory;
153 import org.slf4j.impl.Log4jLoggerAdapter;
155 import org.apache.hbase.thirdparty.com.google.common.io.Closeables;
157 import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
161 * Facility for testing HBase. Replacement for
162 * old HBaseTestCase and HBaseClusterTestCase functionality.
163 * Create an instance and keep it around testing HBase. This class is
164 * meant to be your one-stop shop for anything you might need testing. Manages
165 * one cluster at a time only. Managed cluster can be an in-process
166 * {@link MiniHBaseCluster}, or a deployed cluster of type {@code DistributedHBaseCluster}.
167 * Not all methods work with the real cluster.
168 * Depends on log4j being on classpath and
169 * hbase-site.xml for logging and test-run configuration. It does not set
170 * logging levels.
171 * In the configuration properties, default values for master-info-port and
172 * region-server-port are overridden such that a random port will be assigned (thus
173 * avoiding port contention if another local HBase instance is already running).
174 * <p>To preserve test data directories, pass the system property "hbase.testing.preserve.testdir"
175 * setting it to true.
177 @InterfaceAudience.Public
178 @SuppressWarnings("deprecation")
179 public class HBaseTestingUtility extends HBaseZKTestingUtility {
182 * System property key to get test directory value. Name is as it is because mini dfs has
183 * hard-codings to put test data here. It should NOT be used directly in HBase, as it's a property
184 * used in mini dfs.
185 * @deprecated since 2.0.0 and will be removed in 3.0.0. Can be used only with mini dfs.
186 * @see <a href="https://issues.apache.org/jira/browse/HBASE-19410">HBASE-19410</a>
188 @Deprecated
189 private static final String TEST_DIRECTORY_KEY = "test.build.data";
191 public static final String REGIONS_PER_SERVER_KEY = "hbase.test.regions-per-server";
193 * The default number of regions per regionserver when creating a pre-split
194 * table.
196 public static final int DEFAULT_REGIONS_PER_SERVER = 3;
199 public static final String PRESPLIT_TEST_TABLE_KEY = "hbase.test.pre-split-table";
200 public static final boolean PRESPLIT_TEST_TABLE = true;
202 private MiniDFSCluster dfsCluster = null;
204 private volatile HBaseCluster hbaseCluster = null;
205 private MiniMRCluster mrCluster = null;
207 /** If there is a mini cluster running for this testing utility instance. */
208 private volatile boolean miniClusterRunning;
210 private String hadoopLogDir;
212 /** Directory on test filesystem where we put the data for this instance of
213 * HBaseTestingUtility*/
214 private Path dataTestDirOnTestFS = null;
216 private volatile AsyncClusterConnection asyncConnection;
218 /** Filesystem URI used for map-reduce mini-cluster setup */
219 private static String FS_URI;
221 /** This is for unit tests parameterized with a single boolean. */
222 public static final List<Object[]> MEMSTORETS_TAGS_PARAMETRIZED = memStoreTSAndTagsCombination();
225 * Checks to see if a specific port is available.
227 * @param port the port number to check for availability
228 * @return <tt>true</tt> if the port is available, or <tt>false</tt> if not
230 public static boolean available(int port) {
231 ServerSocket ss = null;
232 DatagramSocket ds = null;
233 try {
234 ss = new ServerSocket(port);
235 ss.setReuseAddress(true);
236 ds = new DatagramSocket(port);
237 ds.setReuseAddress(true);
238 return true;
239 } catch (IOException e) {
240 // Do nothing
241 } finally {
242 if (ds != null) {
243 ds.close();
246 if (ss != null) {
247 try {
248 ss.close();
249 } catch (IOException e) {
250 /* should not be thrown */
255 return false;
259 * Create all combinations of Bloom filters and compression algorithms for
260 * testing.
262 private static List<Object[]> bloomAndCompressionCombinations() {
263 List<Object[]> configurations = new ArrayList<>();
264 for (Compression.Algorithm comprAlgo :
265 HBaseCommonTestingUtility.COMPRESSION_ALGORITHMS) {
266 for (BloomType bloomType : BloomType.values()) {
267 configurations.add(new Object[] { comprAlgo, bloomType });
270 return Collections.unmodifiableList(configurations);
274 * Create combination of memstoreTS and tags
276 private static List<Object[]> memStoreTSAndTagsCombination() {
277 List<Object[]> configurations = new ArrayList<>();
278 configurations.add(new Object[] { false, false });
279 configurations.add(new Object[] { false, true });
280 configurations.add(new Object[] { true, false });
281 configurations.add(new Object[] { true, true });
282 return Collections.unmodifiableList(configurations);
285 public static List<Object[]> memStoreTSTagsAndOffheapCombination() {
286 List<Object[]> configurations = new ArrayList<>();
287 configurations.add(new Object[] { false, false, true });
288 configurations.add(new Object[] { false, false, false });
289 configurations.add(new Object[] { false, true, true });
290 configurations.add(new Object[] { false, true, false });
291 configurations.add(new Object[] { true, false, true });
292 configurations.add(new Object[] { true, false, false });
293 configurations.add(new Object[] { true, true, true });
294 configurations.add(new Object[] { true, true, false });
295 return Collections.unmodifiableList(configurations);
298 public static final Collection<Object[]> BLOOM_AND_COMPRESSION_COMBINATIONS =
299 bloomAndCompressionCombinations();
303 * <p>Create an HBaseTestingUtility using a default configuration.
305 * <p>Initially, all tmp files are written to a local test data directory.
306 * Once {@link #startMiniDFSCluster} is called, either directly or via
307 * {@link #startMiniCluster()}, tmp data will be written to the DFS directory instead.
309 * <p>Previously, there was a distinction between the type of utility returned by
310 * {@link #createLocalHTU()} and this constructor; this is no longer the case. All
311 * HBaseTestingUtility objects will behave as local until a DFS cluster is started,
312 * at which point they will switch to using mini DFS for storage.
314 public HBaseTestingUtility() {
315 this(HBaseConfiguration.create());
319 * <p>Create an HBaseTestingUtility using a given configuration.
321 * <p>Initially, all tmp files are written to a local test data directory.
322 * Once {@link #startMiniDFSCluster} is called, either directly or via
323 * {@link #startMiniCluster()}, tmp data will be written to the DFS directory instead.
325 * <p>Previously, there was a distinction between the type of utility returned by
326 * {@link #createLocalHTU()} and this constructor; this is no longer the case. All
327 * HBaseTestingUtility objects will behave as local until a DFS cluster is started,
328 * at which point they will switch to using mini DFS for storage.
330 * @param conf The configuration to use for further operations
332 public HBaseTestingUtility(@Nullable Configuration conf) {
333 super(conf);
335 // a hbase checksum verification failure will cause unit tests to fail
336 ChecksumUtil.generateExceptionForChecksumFailureForTest(true);
338 // Save this for when setting default file:// breaks things
339 if (this.conf.get("fs.defaultFS") != null) {
340 this.conf.set("original.defaultFS", this.conf.get("fs.defaultFS"));
342 if (this.conf.get(HConstants.HBASE_DIR) != null) {
343 this.conf.set("original.hbase.dir", this.conf.get(HConstants.HBASE_DIR));
345 // Every cluster is a local cluster until we start DFS
346 // Note that conf could be null, but this.conf will not be
347 String dataTestDir = getDataTestDir().toString();
348 this.conf.set("fs.defaultFS","file:///");
349 this.conf.set(HConstants.HBASE_DIR, "file://" + dataTestDir);
350 LOG.debug("Setting {} to {}", HConstants.HBASE_DIR, dataTestDir);
351 this.conf.setBoolean(CommonFSUtils.UNSAFE_STREAM_CAPABILITY_ENFORCE,false);
352 // If the value for random ports isn't set set it to true, thus making
353 // tests opt-out for random port assignment
354 this.conf.setBoolean(LocalHBaseCluster.ASSIGN_RANDOM_PORTS,
355 this.conf.getBoolean(LocalHBaseCluster.ASSIGN_RANDOM_PORTS, true));
359 * @deprecated since 2.0.0 and will be removed in 3.0.0. Use {@link #HBaseTestingUtility()}
360 * instead.
361 * @return a normal HBaseTestingUtility
362 * @see #HBaseTestingUtility()
363 * @see <a href="https://issues.apache.org/jira/browse/HBASE-19841">HBASE-19841</a>
365 @Deprecated
366 public static HBaseTestingUtility createLocalHTU() {
367 return new HBaseTestingUtility();
371 * @deprecated since 2.0.0 and will be removed in 3.0.0. Use
372 * {@link #HBaseTestingUtility(Configuration)} instead.
373 * @return a normal HBaseTestingUtility
374 * @see #HBaseTestingUtility(Configuration)
375 * @see <a href="https://issues.apache.org/jira/browse/HBASE-19841">HBASE-19841</a>
377 @Deprecated
378 public static HBaseTestingUtility createLocalHTU(Configuration c) {
379 return new HBaseTestingUtility(c);
383 * Close both the region {@code r} and it's underlying WAL. For use in tests.
385 public static void closeRegionAndWAL(final Region r) throws IOException {
386 closeRegionAndWAL((HRegion)r);
390 * Close both the HRegion {@code r} and it's underlying WAL. For use in tests.
392 public static void closeRegionAndWAL(final HRegion r) throws IOException {
393 if (r == null) return;
394 r.close();
395 if (r.getWAL() == null) return;
396 r.getWAL().close();
400 * Returns this classes's instance of {@link Configuration}. Be careful how
401 * you use the returned Configuration since {@link Connection} instances
402 * can be shared. The Map of Connections is keyed by the Configuration. If
403 * say, a Connection was being used against a cluster that had been shutdown,
404 * see {@link #shutdownMiniCluster()}, then the Connection will no longer
405 * be wholesome. Rather than use the return direct, its usually best to
406 * make a copy and use that. Do
407 * <code>Configuration c = new Configuration(INSTANCE.getConfiguration());</code>
408 * @return Instance of Configuration.
410 @Override
411 public Configuration getConfiguration() {
412 return super.getConfiguration();
415 public void setHBaseCluster(HBaseCluster hbaseCluster) {
416 this.hbaseCluster = hbaseCluster;
420 * Home our data in a dir under {@link #DEFAULT_BASE_TEST_DIRECTORY}.
421 * Give it a random name so can have many concurrent tests running if
422 * we need to. It needs to amend the {@link #TEST_DIRECTORY_KEY}
423 * System property, as it's what minidfscluster bases
424 * it data dir on. Moding a System property is not the way to do concurrent
425 * instances -- another instance could grab the temporary
426 * value unintentionally -- but not anything can do about it at moment;
427 * single instance only is how the minidfscluster works.
429 * We also create the underlying directory for
430 * hadoop.log.dir, mapreduce.cluster.local.dir and hadoop.tmp.dir, and set the values
431 * in the conf, and as a system property for hadoop.tmp.dir
433 * @return The calculated data test build directory, if newly-created.
435 @Override
436 protected Path setupDataTestDir() {
437 Path testPath = super.setupDataTestDir();
438 if (null == testPath) {
439 return null;
442 createSubDirAndSystemProperty(
443 "hadoop.log.dir",
444 testPath, "hadoop-log-dir");
446 // This is defaulted in core-default.xml to /tmp/hadoop-${user.name}, but
447 // we want our own value to ensure uniqueness on the same machine
448 createSubDirAndSystemProperty(
449 "hadoop.tmp.dir",
450 testPath, "hadoop-tmp-dir");
452 // Read and modified in org.apache.hadoop.mapred.MiniMRCluster
453 createSubDir(
454 "mapreduce.cluster.local.dir",
455 testPath, "mapred-local-dir");
457 return testPath;
460 private void createSubDirAndSystemProperty(
461 String propertyName, Path parent, String subDirName){
463 String sysValue = System.getProperty(propertyName);
465 if (sysValue != null) {
466 // There is already a value set. So we do nothing but hope
467 // that there will be no conflicts
468 LOG.info("System.getProperty(\""+propertyName+"\") already set to: "+
469 sysValue + " so I do NOT create it in " + parent);
470 String confValue = conf.get(propertyName);
471 if (confValue != null && !confValue.endsWith(sysValue)){
472 LOG.warn(
473 propertyName + " property value differs in configuration and system: "+
474 "Configuration="+confValue+" while System="+sysValue+
475 " Erasing configuration value by system value."
478 conf.set(propertyName, sysValue);
479 } else {
480 // Ok, it's not set, so we create it as a subdirectory
481 createSubDir(propertyName, parent, subDirName);
482 System.setProperty(propertyName, conf.get(propertyName));
487 * @return Where to write test data on the test filesystem; Returns working directory
488 * for the test filesystem by default
489 * @see #setupDataTestDirOnTestFS()
490 * @see #getTestFileSystem()
492 private Path getBaseTestDirOnTestFS() throws IOException {
493 FileSystem fs = getTestFileSystem();
494 return new Path(fs.getWorkingDirectory(), "test-data");
498 * Returns a Path in the test filesystem, obtained from {@link #getTestFileSystem()}
499 * to write temporary test data. Call this method after setting up the mini dfs cluster
500 * if the test relies on it.
501 * @return a unique path in the test filesystem
503 public Path getDataTestDirOnTestFS() throws IOException {
504 if (dataTestDirOnTestFS == null) {
505 setupDataTestDirOnTestFS();
508 return dataTestDirOnTestFS;
512 * Returns a Path in the test filesystem, obtained from {@link #getTestFileSystem()}
513 * to write temporary test data. Call this method after setting up the mini dfs cluster
514 * if the test relies on it.
515 * @return a unique path in the test filesystem
516 * @param subdirName name of the subdir to create under the base test dir
518 public Path getDataTestDirOnTestFS(final String subdirName) throws IOException {
519 return new Path(getDataTestDirOnTestFS(), subdirName);
523 * Sets up a path in test filesystem to be used by tests.
524 * Creates a new directory if not already setup.
526 private void setupDataTestDirOnTestFS() throws IOException {
527 if (dataTestDirOnTestFS != null) {
528 LOG.warn("Data test on test fs dir already setup in "
529 + dataTestDirOnTestFS.toString());
530 return;
532 dataTestDirOnTestFS = getNewDataTestDirOnTestFS();
536 * Sets up a new path in test filesystem to be used by tests.
538 private Path getNewDataTestDirOnTestFS() throws IOException {
539 //The file system can be either local, mini dfs, or if the configuration
540 //is supplied externally, it can be an external cluster FS. If it is a local
541 //file system, the tests should use getBaseTestDir, otherwise, we can use
542 //the working directory, and create a unique sub dir there
543 FileSystem fs = getTestFileSystem();
544 Path newDataTestDir;
545 String randomStr = getRandomUUID().toString();
546 if (fs.getUri().getScheme().equals(FileSystem.getLocal(conf).getUri().getScheme())) {
547 newDataTestDir = new Path(getDataTestDir(), randomStr);
548 File dataTestDir = new File(newDataTestDir.toString());
549 if (deleteOnExit()) dataTestDir.deleteOnExit();
550 } else {
551 Path base = getBaseTestDirOnTestFS();
552 newDataTestDir = new Path(base, randomStr);
553 if (deleteOnExit()) fs.deleteOnExit(newDataTestDir);
555 return newDataTestDir;
559 * Cleans the test data directory on the test filesystem.
560 * @return True if we removed the test dirs
561 * @throws IOException
563 public boolean cleanupDataTestDirOnTestFS() throws IOException {
564 boolean ret = getTestFileSystem().delete(dataTestDirOnTestFS, true);
565 if (ret)
566 dataTestDirOnTestFS = null;
567 return ret;
571 * Cleans a subdirectory under the test data directory on the test filesystem.
572 * @return True if we removed child
573 * @throws IOException
575 public boolean cleanupDataTestDirOnTestFS(String subdirName) throws IOException {
576 Path cpath = getDataTestDirOnTestFS(subdirName);
577 return getTestFileSystem().delete(cpath, true);
581 * Start a minidfscluster.
582 * @param servers How many DNs to start.
583 * @throws Exception
584 * @see #shutdownMiniDFSCluster()
585 * @return The mini dfs cluster created.
587 public MiniDFSCluster startMiniDFSCluster(int servers) throws Exception {
588 return startMiniDFSCluster(servers, null);
592 * Start a minidfscluster.
593 * This is useful if you want to run datanode on distinct hosts for things
594 * like HDFS block location verification.
595 * If you start MiniDFSCluster without host names, all instances of the
596 * datanodes will have the same host name.
597 * @param hosts hostnames DNs to run on.
598 * @throws Exception
599 * @see #shutdownMiniDFSCluster()
600 * @return The mini dfs cluster created.
602 public MiniDFSCluster startMiniDFSCluster(final String hosts[])
603 throws Exception {
604 if ( hosts != null && hosts.length != 0) {
605 return startMiniDFSCluster(hosts.length, hosts);
606 } else {
607 return startMiniDFSCluster(1, null);
612 * Start a minidfscluster.
613 * Can only create one.
614 * @param servers How many DNs to start.
615 * @param hosts hostnames DNs to run on.
616 * @throws Exception
617 * @see #shutdownMiniDFSCluster()
618 * @return The mini dfs cluster created.
620 public MiniDFSCluster startMiniDFSCluster(int servers, final String hosts[])
621 throws Exception {
622 return startMiniDFSCluster(servers, null, hosts);
625 private void setFs() throws IOException {
626 if(this.dfsCluster == null){
627 LOG.info("Skipping setting fs because dfsCluster is null");
628 return;
630 FileSystem fs = this.dfsCluster.getFileSystem();
631 FSUtils.setFsDefault(this.conf, new Path(fs.getUri()));
633 // re-enable this check with dfs
634 conf.unset(CommonFSUtils.UNSAFE_STREAM_CAPABILITY_ENFORCE);
637 public MiniDFSCluster startMiniDFSCluster(int servers, final String racks[], String hosts[])
638 throws Exception {
639 createDirsAndSetProperties();
640 EditLogFileOutputStream.setShouldSkipFsyncForTesting(true);
642 // Error level to skip some warnings specific to the minicluster. See HBASE-4709
643 org.apache.log4j.Logger.getLogger(org.apache.hadoop.metrics2.util.MBeans.class).
644 setLevel(org.apache.log4j.Level.ERROR);
645 org.apache.log4j.Logger.getLogger(org.apache.hadoop.metrics2.impl.MetricsSystemImpl.class).
646 setLevel(org.apache.log4j.Level.ERROR);
648 TraceUtil.initTracer(conf);
650 this.dfsCluster = new MiniDFSCluster(0, this.conf, servers, true, true,
651 true, null, racks, hosts, null);
653 // Set this just-started cluster as our filesystem.
654 setFs();
656 // Wait for the cluster to be totally up
657 this.dfsCluster.waitClusterUp();
659 //reset the test directory for test file system
660 dataTestDirOnTestFS = null;
661 String dataTestDir = getDataTestDir().toString();
662 conf.set(HConstants.HBASE_DIR, dataTestDir);
663 LOG.debug("Setting {} to {}", HConstants.HBASE_DIR, dataTestDir);
665 return this.dfsCluster;
668 public MiniDFSCluster startMiniDFSClusterForTestWAL(int namenodePort) throws IOException {
669 createDirsAndSetProperties();
670 dfsCluster = new MiniDFSCluster(namenodePort, conf, 5, false, true, true, null,
671 null, null, null);
672 return dfsCluster;
675 /** This is used before starting HDFS and map-reduce mini-clusters */
676 private void createDirsAndSetProperties() throws IOException {
677 setupClusterTestDir();
678 conf.set(TEST_DIRECTORY_KEY, clusterTestDir.getPath());
679 System.setProperty(TEST_DIRECTORY_KEY, clusterTestDir.getPath());
680 createDirAndSetProperty("cache_data", "test.cache.data");
681 createDirAndSetProperty("hadoop_tmp", "hadoop.tmp.dir");
682 hadoopLogDir = createDirAndSetProperty("hadoop_logs", "hadoop.log.dir");
683 createDirAndSetProperty("mapred_local", "mapreduce.cluster.local.dir");
684 createDirAndSetProperty("mapred_temp", "mapreduce.cluster.temp.dir");
685 enableShortCircuit();
687 Path root = getDataTestDirOnTestFS("hadoop");
688 conf.set(MapreduceTestingShim.getMROutputDirProp(),
689 new Path(root, "mapred-output-dir").toString());
690 conf.set("mapreduce.jobtracker.system.dir", new Path(root, "mapred-system-dir").toString());
691 conf.set("mapreduce.jobtracker.staging.root.dir",
692 new Path(root, "mapreduce-jobtracker-staging-root-dir").toString());
693 conf.set("mapreduce.job.working.dir", new Path(root, "mapred-working-dir").toString());
694 conf.set("yarn.app.mapreduce.am.staging-dir",
695 new Path(root, "mapreduce-am-staging-root-dir").toString());
699 * Check whether the tests should assume NEW_VERSION_BEHAVIOR when creating
700 * new column families. Default to false.
702 public boolean isNewVersionBehaviorEnabled(){
703 final String propName = "hbase.tests.new.version.behavior";
704 String v = System.getProperty(propName);
705 if (v != null){
706 return Boolean.parseBoolean(v);
708 return false;
712 * Get the HBase setting for dfs.client.read.shortcircuit from the conf or a system property.
713 * This allows to specify this parameter on the command line.
714 * If not set, default is true.
716 public boolean isReadShortCircuitOn(){
717 final String propName = "hbase.tests.use.shortcircuit.reads";
718 String readOnProp = System.getProperty(propName);
719 if (readOnProp != null){
720 return Boolean.parseBoolean(readOnProp);
721 } else {
722 return conf.getBoolean(propName, false);
726 /** Enable the short circuit read, unless configured differently.
727 * Set both HBase and HDFS settings, including skipping the hdfs checksum checks.
729 private void enableShortCircuit() {
730 if (isReadShortCircuitOn()) {
731 String curUser = System.getProperty("user.name");
732 LOG.info("read short circuit is ON for user " + curUser);
733 // read short circuit, for hdfs
734 conf.set("dfs.block.local-path-access.user", curUser);
735 // read short circuit, for hbase
736 conf.setBoolean("dfs.client.read.shortcircuit", true);
737 // Skip checking checksum, for the hdfs client and the datanode
738 conf.setBoolean("dfs.client.read.shortcircuit.skip.checksum", true);
739 } else {
740 LOG.info("read short circuit is OFF");
744 private String createDirAndSetProperty(final String relPath, String property) {
745 String path = getDataTestDir(relPath).toString();
746 System.setProperty(property, path);
747 conf.set(property, path);
748 new File(path).mkdirs();
749 LOG.info("Setting " + property + " to " + path + " in system properties and HBase conf");
750 return path;
754 * Shuts down instance created by call to {@link #startMiniDFSCluster(int)}
755 * or does nothing.
756 * @throws IOException
758 public void shutdownMiniDFSCluster() throws IOException {
759 if (this.dfsCluster != null) {
760 // The below throws an exception per dn, AsynchronousCloseException.
761 this.dfsCluster.shutdown();
762 dfsCluster = null;
763 dataTestDirOnTestFS = null;
764 FSUtils.setFsDefault(this.conf, new Path("file:///"));
769 * Start up a minicluster of hbase, dfs, and zookeeper where WAL's walDir is created separately.
770 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
771 * @param createWALDir Whether to create a new WAL directory.
772 * @return The mini HBase cluster created.
773 * @see #shutdownMiniCluster()
774 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
775 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
776 * @see #startMiniCluster(StartMiniClusterOption)
777 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
779 @Deprecated
780 public MiniHBaseCluster startMiniCluster(boolean createWALDir) throws Exception {
781 StartMiniClusterOption option = StartMiniClusterOption.builder()
782 .createWALDir(createWALDir).build();
783 return startMiniCluster(option);
787 * Start up a minicluster of hbase, dfs, and zookeeper.
788 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
789 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
790 * @param createRootDir Whether to create a new root or data directory path.
791 * @return The mini HBase cluster created.
792 * @see #shutdownMiniCluster()
793 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
794 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
795 * @see #startMiniCluster(StartMiniClusterOption)
796 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
798 @Deprecated
799 public MiniHBaseCluster startMiniCluster(int numSlaves, boolean createRootDir)
800 throws Exception {
801 StartMiniClusterOption option = StartMiniClusterOption.builder()
802 .numRegionServers(numSlaves).numDataNodes(numSlaves).createRootDir(createRootDir).build();
803 return startMiniCluster(option);
807 * Start up a minicluster of hbase, dfs, and zookeeper.
808 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
809 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
810 * @param createRootDir Whether to create a new root or data directory path.
811 * @param createWALDir Whether to create a new WAL directory.
812 * @return The mini HBase cluster created.
813 * @see #shutdownMiniCluster()
814 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
815 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
816 * @see #startMiniCluster(StartMiniClusterOption)
817 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
819 @Deprecated
820 public MiniHBaseCluster startMiniCluster(int numSlaves, boolean createRootDir,
821 boolean createWALDir) throws Exception {
822 StartMiniClusterOption option = StartMiniClusterOption.builder()
823 .numRegionServers(numSlaves).numDataNodes(numSlaves).createRootDir(createRootDir)
824 .createWALDir(createWALDir).build();
825 return startMiniCluster(option);
829 * Start up a minicluster of hbase, dfs, and zookeeper.
830 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
831 * @param numMasters Master node number.
832 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
833 * @param createRootDir Whether to create a new root or data directory path.
834 * @return The mini HBase cluster created.
835 * @see #shutdownMiniCluster()
836 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
837 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
838 * @see #startMiniCluster(StartMiniClusterOption)
839 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
841 @Deprecated
842 public MiniHBaseCluster startMiniCluster(int numMasters, int numSlaves, boolean createRootDir)
843 throws Exception {
844 StartMiniClusterOption option = StartMiniClusterOption.builder()
845 .numMasters(numMasters).numRegionServers(numSlaves).createRootDir(createRootDir)
846 .numDataNodes(numSlaves).build();
847 return startMiniCluster(option);
851 * Start up a minicluster of hbase, dfs, and zookeeper.
852 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
853 * @param numMasters Master node number.
854 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
855 * @return The mini HBase cluster created.
856 * @see #shutdownMiniCluster()
857 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
858 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
859 * @see #startMiniCluster(StartMiniClusterOption)
860 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
862 @Deprecated
863 public MiniHBaseCluster startMiniCluster(int numMasters, int numSlaves) throws Exception {
864 StartMiniClusterOption option = StartMiniClusterOption.builder()
865 .numMasters(numMasters).numRegionServers(numSlaves).numDataNodes(numSlaves).build();
866 return startMiniCluster(option);
870 * Start up a minicluster of hbase, dfs, and zookeeper.
871 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
872 * @param numMasters Master node number.
873 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
874 * @param dataNodeHosts The hostnames of DataNodes to run on. If not null, its size will overwrite
875 * HDFS data node number.
876 * @param createRootDir Whether to create a new root or data directory path.
877 * @return The mini HBase cluster created.
878 * @see #shutdownMiniCluster()
879 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
880 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
881 * @see #startMiniCluster(StartMiniClusterOption)
882 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
884 @Deprecated
885 public MiniHBaseCluster startMiniCluster(int numMasters, int numSlaves, String[] dataNodeHosts,
886 boolean createRootDir) throws Exception {
887 StartMiniClusterOption option = StartMiniClusterOption.builder()
888 .numMasters(numMasters).numRegionServers(numSlaves).createRootDir(createRootDir)
889 .numDataNodes(numSlaves).dataNodeHosts(dataNodeHosts).build();
890 return startMiniCluster(option);
894 * Start up a minicluster of hbase, dfs, and zookeeper.
895 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
896 * @param numMasters Master node number.
897 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
898 * @param dataNodeHosts The hostnames of DataNodes to run on. If not null, its size will overwrite
899 * HDFS data node number.
900 * @return The mini HBase cluster created.
901 * @see #shutdownMiniCluster()
902 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
903 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
904 * @see #startMiniCluster(StartMiniClusterOption)
905 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
907 @Deprecated
908 public MiniHBaseCluster startMiniCluster(int numMasters, int numSlaves, String[] dataNodeHosts)
909 throws Exception {
910 StartMiniClusterOption option = StartMiniClusterOption.builder()
911 .numMasters(numMasters).numRegionServers(numSlaves)
912 .numDataNodes(numSlaves).dataNodeHosts(dataNodeHosts).build();
913 return startMiniCluster(option);
917 * Start up a minicluster of hbase, dfs, and zookeeper.
918 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
919 * @param numMasters Master node number.
920 * @param numRegionServers Number of region servers.
921 * @param numDataNodes Number of datanodes.
922 * @return The mini HBase cluster created.
923 * @see #shutdownMiniCluster()
924 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
925 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
926 * @see #startMiniCluster(StartMiniClusterOption)
927 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
929 @Deprecated
930 public MiniHBaseCluster startMiniCluster(int numMasters, int numRegionServers, int numDataNodes)
931 throws Exception {
932 StartMiniClusterOption option = StartMiniClusterOption.builder()
933 .numMasters(numMasters).numRegionServers(numRegionServers).numDataNodes(numDataNodes)
934 .build();
935 return startMiniCluster(option);
939 * Start up a minicluster of hbase, dfs, and zookeeper.
940 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
941 * @param numMasters Master node number.
942 * @param numSlaves Slave node number, for both HBase region server and HDFS data node.
943 * @param dataNodeHosts The hostnames of DataNodes to run on. If not null, its size will overwrite
944 * HDFS data node number.
945 * @param masterClass The class to use as HMaster, or null for default.
946 * @param rsClass The class to use as HRegionServer, or null for default.
947 * @return The mini HBase cluster created.
948 * @see #shutdownMiniCluster()
949 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
950 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
951 * @see #startMiniCluster(StartMiniClusterOption)
952 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
954 @Deprecated
955 public MiniHBaseCluster startMiniCluster(int numMasters, int numSlaves, String[] dataNodeHosts,
956 Class<? extends HMaster> masterClass,
957 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> rsClass)
958 throws Exception {
959 StartMiniClusterOption option = StartMiniClusterOption.builder()
960 .numMasters(numMasters).masterClass(masterClass)
961 .numRegionServers(numSlaves).rsClass(rsClass)
962 .numDataNodes(numSlaves).dataNodeHosts(dataNodeHosts)
963 .build();
964 return startMiniCluster(option);
968 * Start up a minicluster of hbase, dfs, and zookeeper.
969 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
970 * @param numMasters Master node number.
971 * @param numRegionServers Number of region servers.
972 * @param numDataNodes Number of datanodes.
973 * @param dataNodeHosts The hostnames of DataNodes to run on. If not null, its size will overwrite
974 * HDFS data node number.
975 * @param masterClass The class to use as HMaster, or null for default.
976 * @param rsClass The class to use as HRegionServer, or null for default.
977 * @return The mini HBase cluster created.
978 * @see #shutdownMiniCluster()
979 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
980 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
981 * @see #startMiniCluster(StartMiniClusterOption)
982 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
984 @Deprecated
985 public MiniHBaseCluster startMiniCluster(int numMasters, int numRegionServers, int numDataNodes,
986 String[] dataNodeHosts, Class<? extends HMaster> masterClass,
987 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> rsClass)
988 throws Exception {
989 StartMiniClusterOption option = StartMiniClusterOption.builder()
990 .numMasters(numMasters).masterClass(masterClass)
991 .numRegionServers(numRegionServers).rsClass(rsClass)
992 .numDataNodes(numDataNodes).dataNodeHosts(dataNodeHosts)
993 .build();
994 return startMiniCluster(option);
998 * Start up a minicluster of hbase, dfs, and zookeeper.
999 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
1000 * @param numMasters Master node number.
1001 * @param numRegionServers Number of region servers.
1002 * @param numDataNodes Number of datanodes.
1003 * @param dataNodeHosts The hostnames of DataNodes to run on. If not null, its size will overwrite
1004 * HDFS data node number.
1005 * @param masterClass The class to use as HMaster, or null for default.
1006 * @param rsClass The class to use as HRegionServer, or null for default.
1007 * @param createRootDir Whether to create a new root or data directory path.
1008 * @param createWALDir Whether to create a new WAL directory.
1009 * @return The mini HBase cluster created.
1010 * @see #shutdownMiniCluster()
1011 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
1012 * {@link #startMiniCluster(StartMiniClusterOption)} instead.
1013 * @see #startMiniCluster(StartMiniClusterOption)
1014 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
1016 @Deprecated
1017 public MiniHBaseCluster startMiniCluster(int numMasters, int numRegionServers, int numDataNodes,
1018 String[] dataNodeHosts, Class<? extends HMaster> masterClass,
1019 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> rsClass, boolean createRootDir,
1020 boolean createWALDir) throws Exception {
1021 StartMiniClusterOption option = StartMiniClusterOption.builder()
1022 .numMasters(numMasters).masterClass(masterClass)
1023 .numRegionServers(numRegionServers).rsClass(rsClass)
1024 .numDataNodes(numDataNodes).dataNodeHosts(dataNodeHosts)
1025 .createRootDir(createRootDir).createWALDir(createWALDir)
1026 .build();
1027 return startMiniCluster(option);
1031 * Start up a minicluster of hbase, dfs and zookeeper clusters with given slave node number.
1032 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
1033 * @param numSlaves slave node number, for both HBase region server and HDFS data node.
1034 * @see #startMiniCluster(StartMiniClusterOption option)
1035 * @see #shutdownMiniDFSCluster()
1037 public MiniHBaseCluster startMiniCluster(int numSlaves) throws Exception {
1038 StartMiniClusterOption option = StartMiniClusterOption.builder()
1039 .numRegionServers(numSlaves).numDataNodes(numSlaves).build();
1040 return startMiniCluster(option);
1044 * Start up a minicluster of hbase, dfs and zookeeper all using default options.
1045 * Option default value can be found in {@link StartMiniClusterOption.Builder}.
1046 * @see #startMiniCluster(StartMiniClusterOption option)
1047 * @see #shutdownMiniDFSCluster()
1049 public MiniHBaseCluster startMiniCluster() throws Exception {
1050 return startMiniCluster(StartMiniClusterOption.builder().build());
1054 * Start up a mini cluster of hbase, optionally dfs and zookeeper if needed.
1055 * It modifies Configuration. It homes the cluster data directory under a random
1056 * subdirectory in a directory under System property test.build.data, to be cleaned up on exit.
1057 * @see #shutdownMiniDFSCluster()
1059 public MiniHBaseCluster startMiniCluster(StartMiniClusterOption option) throws Exception {
1060 LOG.info("Starting up minicluster with option: {}", option);
1062 // If we already put up a cluster, fail.
1063 if (miniClusterRunning) {
1064 throw new IllegalStateException("A mini-cluster is already running");
1066 miniClusterRunning = true;
1068 setupClusterTestDir();
1069 System.setProperty(TEST_DIRECTORY_KEY, this.clusterTestDir.getPath());
1071 // Bring up mini dfs cluster. This spews a bunch of warnings about missing
1072 // scheme. Complaints are 'Scheme is undefined for build/test/data/dfs/name1'.
1073 if (dfsCluster == null) {
1074 LOG.info("STARTING DFS");
1075 dfsCluster = startMiniDFSCluster(option.getNumDataNodes(), option.getDataNodeHosts());
1076 } else {
1077 LOG.info("NOT STARTING DFS");
1080 // Start up a zk cluster.
1081 if (getZkCluster() == null) {
1082 startMiniZKCluster(option.getNumZkServers());
1085 // Start the MiniHBaseCluster
1086 return startMiniHBaseCluster(option);
1090 * Starts up mini hbase cluster.
1091 * Usually you won't want this. You'll usually want {@link #startMiniCluster()}.
1092 * This is useful when doing stepped startup of clusters.
1093 * @return Reference to the hbase mini hbase cluster.
1094 * @see #startMiniCluster(StartMiniClusterOption)
1095 * @see #shutdownMiniHBaseCluster()
1097 public MiniHBaseCluster startMiniHBaseCluster(StartMiniClusterOption option)
1098 throws IOException, InterruptedException {
1099 // Now do the mini hbase cluster. Set the hbase.rootdir in config.
1100 createRootDir(option.isCreateRootDir());
1101 if (option.isCreateWALDir()) {
1102 createWALRootDir();
1104 // Set the hbase.fs.tmp.dir config to make sure that we have some default value. This is
1105 // for tests that do not read hbase-defaults.xml
1106 setHBaseFsTmpDir();
1108 // These settings will make the server waits until this exact number of
1109 // regions servers are connected.
1110 if (conf.getInt(ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART, -1) == -1) {
1111 conf.setInt(ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART, option.getNumRegionServers());
1113 if (conf.getInt(ServerManager.WAIT_ON_REGIONSERVERS_MAXTOSTART, -1) == -1) {
1114 conf.setInt(ServerManager.WAIT_ON_REGIONSERVERS_MAXTOSTART, option.getNumRegionServers());
1117 Configuration c = new Configuration(this.conf);
1118 TraceUtil.initTracer(c);
1119 this.hbaseCluster =
1120 new MiniHBaseCluster(c, option.getNumMasters(), option.getNumRegionServers(),
1121 option.getRsPorts(), option.getMasterClass(), option.getRsClass());
1122 // Don't leave here till we've done a successful scan of the hbase:meta
1123 Table t = getConnection().getTable(TableName.META_TABLE_NAME);
1124 ResultScanner s = t.getScanner(new Scan());
1125 while (s.next() != null) {
1126 continue;
1128 s.close();
1129 t.close();
1131 getAdmin(); // create immediately the hbaseAdmin
1132 LOG.info("Minicluster is up; activeMaster={}", getHBaseCluster().getMaster());
1134 return (MiniHBaseCluster) hbaseCluster;
1138 * Starts up mini hbase cluster using default options.
1139 * Default options can be found in {@link StartMiniClusterOption.Builder}.
1140 * @see #startMiniHBaseCluster(StartMiniClusterOption)
1141 * @see #shutdownMiniHBaseCluster()
1143 public MiniHBaseCluster startMiniHBaseCluster() throws IOException, InterruptedException {
1144 return startMiniHBaseCluster(StartMiniClusterOption.builder().build());
1148 * Starts up mini hbase cluster.
1149 * Usually you won't want this. You'll usually want {@link #startMiniCluster()}.
1150 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
1151 * @param numMasters Master node number.
1152 * @param numRegionServers Number of region servers.
1153 * @return The mini HBase cluster created.
1154 * @see #shutdownMiniHBaseCluster()
1155 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
1156 * {@link #startMiniHBaseCluster(StartMiniClusterOption)} instead.
1157 * @see #startMiniHBaseCluster(StartMiniClusterOption)
1158 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
1160 @Deprecated
1161 public MiniHBaseCluster startMiniHBaseCluster(int numMasters, int numRegionServers)
1162 throws IOException, InterruptedException {
1163 StartMiniClusterOption option = StartMiniClusterOption.builder()
1164 .numMasters(numMasters).numRegionServers(numRegionServers).build();
1165 return startMiniHBaseCluster(option);
1169 * Starts up mini hbase cluster.
1170 * Usually you won't want this. You'll usually want {@link #startMiniCluster()}.
1171 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
1172 * @param numMasters Master node number.
1173 * @param numRegionServers Number of region servers.
1174 * @param rsPorts Ports that RegionServer should use.
1175 * @return The mini HBase cluster created.
1176 * @see #shutdownMiniHBaseCluster()
1177 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
1178 * {@link #startMiniHBaseCluster(StartMiniClusterOption)} instead.
1179 * @see #startMiniHBaseCluster(StartMiniClusterOption)
1180 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
1182 @Deprecated
1183 public MiniHBaseCluster startMiniHBaseCluster(int numMasters, int numRegionServers,
1184 List<Integer> rsPorts) throws IOException, InterruptedException {
1185 StartMiniClusterOption option = StartMiniClusterOption.builder()
1186 .numMasters(numMasters).numRegionServers(numRegionServers).rsPorts(rsPorts).build();
1187 return startMiniHBaseCluster(option);
1191 * Starts up mini hbase cluster.
1192 * Usually you won't want this. You'll usually want {@link #startMiniCluster()}.
1193 * All other options will use default values, defined in {@link StartMiniClusterOption.Builder}.
1194 * @param numMasters Master node number.
1195 * @param numRegionServers Number of region servers.
1196 * @param rsPorts Ports that RegionServer should use.
1197 * @param masterClass The class to use as HMaster, or null for default.
1198 * @param rsClass The class to use as HRegionServer, or null for default.
1199 * @param createRootDir Whether to create a new root or data directory path.
1200 * @param createWALDir Whether to create a new WAL directory.
1201 * @return The mini HBase cluster created.
1202 * @see #shutdownMiniHBaseCluster()
1203 * @deprecated since 2.2.0 and will be removed in 4.0.0. Use
1204 * {@link #startMiniHBaseCluster(StartMiniClusterOption)} instead.
1205 * @see #startMiniHBaseCluster(StartMiniClusterOption)
1206 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21071">HBASE-21071</a>
1208 @Deprecated
1209 public MiniHBaseCluster startMiniHBaseCluster(int numMasters, int numRegionServers,
1210 List<Integer> rsPorts, Class<? extends HMaster> masterClass,
1211 Class<? extends MiniHBaseCluster.MiniHBaseClusterRegionServer> rsClass,
1212 boolean createRootDir, boolean createWALDir) throws IOException, InterruptedException {
1213 StartMiniClusterOption option = StartMiniClusterOption.builder()
1214 .numMasters(numMasters).masterClass(masterClass)
1215 .numRegionServers(numRegionServers).rsClass(rsClass).rsPorts(rsPorts)
1216 .createRootDir(createRootDir).createWALDir(createWALDir).build();
1217 return startMiniHBaseCluster(option);
1221 * Starts the hbase cluster up again after shutting it down previously in a
1222 * test. Use this if you want to keep dfs/zk up and just stop/start hbase.
1223 * @param servers number of region servers
1225 public void restartHBaseCluster(int servers) throws IOException, InterruptedException {
1226 this.restartHBaseCluster(servers, null);
1229 public void restartHBaseCluster(int servers, List<Integer> ports)
1230 throws IOException, InterruptedException {
1231 StartMiniClusterOption option =
1232 StartMiniClusterOption.builder().numRegionServers(servers).rsPorts(ports).build();
1233 restartHBaseCluster(option);
1236 public void restartHBaseCluster(StartMiniClusterOption option)
1237 throws IOException, InterruptedException {
1238 if (hbaseAdmin != null) {
1239 hbaseAdmin.close();
1240 hbaseAdmin = null;
1242 if (this.asyncConnection != null) {
1243 this.asyncConnection.close();
1244 this.asyncConnection = null;
1246 this.hbaseCluster =
1247 new MiniHBaseCluster(this.conf, option.getNumMasters(), option.getNumRegionServers(),
1248 option.getRsPorts(), option.getMasterClass(), option.getRsClass());
1249 // Don't leave here till we've done a successful scan of the hbase:meta
1250 Connection conn = ConnectionFactory.createConnection(this.conf);
1251 Table t = conn.getTable(TableName.META_TABLE_NAME);
1252 ResultScanner s = t.getScanner(new Scan());
1253 while (s.next() != null) {
1254 // do nothing
1256 LOG.info("HBase has been restarted");
1257 s.close();
1258 t.close();
1259 conn.close();
1263 * @return Current mini hbase cluster. Only has something in it after a call
1264 * to {@link #startMiniCluster()}.
1265 * @see #startMiniCluster()
1267 public MiniHBaseCluster getMiniHBaseCluster() {
1268 if (this.hbaseCluster == null || this.hbaseCluster instanceof MiniHBaseCluster) {
1269 return (MiniHBaseCluster)this.hbaseCluster;
1271 throw new RuntimeException(hbaseCluster + " not an instance of " +
1272 MiniHBaseCluster.class.getName());
1276 * Stops mini hbase, zk, and hdfs clusters.
1277 * @throws IOException
1278 * @see #startMiniCluster(int)
1280 public void shutdownMiniCluster() throws Exception {
1281 LOG.info("Shutting down minicluster");
1282 shutdownMiniHBaseCluster();
1283 shutdownMiniDFSCluster();
1284 shutdownMiniZKCluster();
1286 cleanupTestDir();
1287 miniClusterRunning = false;
1288 LOG.info("Minicluster is down");
1292 * Shutdown HBase mini cluster.Does not shutdown zk or dfs if running.
1293 * @throws java.io.IOException in case command is unsuccessful
1295 public void shutdownMiniHBaseCluster() throws IOException {
1296 cleanup();
1297 if (this.hbaseCluster != null) {
1298 this.hbaseCluster.shutdown();
1299 // Wait till hbase is down before going on to shutdown zk.
1300 this.hbaseCluster.waitUntilShutDown();
1301 this.hbaseCluster = null;
1303 if (zooKeeperWatcher != null) {
1304 zooKeeperWatcher.close();
1305 zooKeeperWatcher = null;
1310 * Abruptly Shutdown HBase mini cluster. Does not shutdown zk or dfs if running.
1311 * @throws java.io.IOException throws in case command is unsuccessful
1313 public void killMiniHBaseCluster() throws IOException {
1314 cleanup();
1315 if (this.hbaseCluster != null) {
1316 getMiniHBaseCluster().killAll();
1317 this.hbaseCluster = null;
1319 if (zooKeeperWatcher != null) {
1320 zooKeeperWatcher.close();
1321 zooKeeperWatcher = null;
1325 // close hbase admin, close current connection and reset MIN MAX configs for RS.
1326 private void cleanup() throws IOException {
1327 closeConnection();
1328 // unset the configuration for MIN and MAX RS to start
1329 conf.setInt(ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART, -1);
1330 conf.setInt(ServerManager.WAIT_ON_REGIONSERVERS_MAXTOSTART, -1);
1334 * Returns the path to the default root dir the minicluster uses. If <code>create</code>
1335 * is true, a new root directory path is fetched irrespective of whether it has been fetched
1336 * before or not. If false, previous path is used.
1337 * Note: this does not cause the root dir to be created.
1338 * @return Fully qualified path for the default hbase root dir
1339 * @throws IOException
1341 public Path getDefaultRootDirPath(boolean create) throws IOException {
1342 if (!create) {
1343 return getDataTestDirOnTestFS();
1344 } else {
1345 return getNewDataTestDirOnTestFS();
1350 * Same as {{@link HBaseTestingUtility#getDefaultRootDirPath(boolean create)}
1351 * except that <code>create</code> flag is false.
1352 * Note: this does not cause the root dir to be created.
1353 * @return Fully qualified path for the default hbase root dir
1354 * @throws IOException
1356 public Path getDefaultRootDirPath() throws IOException {
1357 return getDefaultRootDirPath(false);
1361 * Creates an hbase rootdir in user home directory. Also creates hbase
1362 * version file. Normally you won't make use of this method. Root hbasedir
1363 * is created for you as part of mini cluster startup. You'd only use this
1364 * method if you were doing manual operation.
1365 * @param create This flag decides whether to get a new
1366 * root or data directory path or not, if it has been fetched already.
1367 * Note : Directory will be made irrespective of whether path has been fetched or not.
1368 * If directory already exists, it will be overwritten
1369 * @return Fully qualified path to hbase root dir
1370 * @throws IOException
1372 public Path createRootDir(boolean create) throws IOException {
1373 FileSystem fs = FileSystem.get(this.conf);
1374 Path hbaseRootdir = getDefaultRootDirPath(create);
1375 FSUtils.setRootDir(this.conf, hbaseRootdir);
1376 fs.mkdirs(hbaseRootdir);
1377 FSUtils.setVersion(fs, hbaseRootdir);
1378 return hbaseRootdir;
1382 * Same as {@link HBaseTestingUtility#createRootDir(boolean create)}
1383 * except that <code>create</code> flag is false.
1384 * @return Fully qualified path to hbase root dir
1385 * @throws IOException
1387 public Path createRootDir() throws IOException {
1388 return createRootDir(false);
1392 * Creates a hbase walDir in the user's home directory.
1393 * Normally you won't make use of this method. Root hbaseWALDir
1394 * is created for you as part of mini cluster startup. You'd only use this
1395 * method if you were doing manual operation.
1397 * @return Fully qualified path to hbase root dir
1398 * @throws IOException
1400 public Path createWALRootDir() throws IOException {
1401 FileSystem fs = FileSystem.get(this.conf);
1402 Path walDir = getNewDataTestDirOnTestFS();
1403 FSUtils.setWALRootDir(this.conf, walDir);
1404 fs.mkdirs(walDir);
1405 return walDir;
1408 private void setHBaseFsTmpDir() throws IOException {
1409 String hbaseFsTmpDirInString = this.conf.get("hbase.fs.tmp.dir");
1410 if (hbaseFsTmpDirInString == null) {
1411 this.conf.set("hbase.fs.tmp.dir", getDataTestDirOnTestFS("hbase-staging").toString());
1412 LOG.info("Setting hbase.fs.tmp.dir to " + this.conf.get("hbase.fs.tmp.dir"));
1413 } else {
1414 LOG.info("The hbase.fs.tmp.dir is set to " + hbaseFsTmpDirInString);
1419 * Flushes all caches in the mini hbase cluster
1420 * @throws IOException
1422 public void flush() throws IOException {
1423 getMiniHBaseCluster().flushcache();
1427 * Flushes all caches in the mini hbase cluster
1428 * @throws IOException
1430 public void flush(TableName tableName) throws IOException {
1431 getMiniHBaseCluster().flushcache(tableName);
1435 * Compact all regions in the mini hbase cluster
1436 * @throws IOException
1438 public void compact(boolean major) throws IOException {
1439 getMiniHBaseCluster().compact(major);
1443 * Compact all of a table's reagion in the mini hbase cluster
1444 * @throws IOException
1446 public void compact(TableName tableName, boolean major) throws IOException {
1447 getMiniHBaseCluster().compact(tableName, major);
1451 * Create a table.
1452 * @param tableName
1453 * @param family
1454 * @return A Table instance for the created table.
1455 * @throws IOException
1457 public Table createTable(TableName tableName, String family)
1458 throws IOException{
1459 return createTable(tableName, new String[]{family});
1463 * Create a table.
1464 * @param tableName
1465 * @param families
1466 * @return A Table instance for the created table.
1467 * @throws IOException
1469 public Table createTable(TableName tableName, String[] families)
1470 throws IOException {
1471 List<byte[]> fams = new ArrayList<>(families.length);
1472 for (String family : families) {
1473 fams.add(Bytes.toBytes(family));
1475 return createTable(tableName, fams.toArray(new byte[0][]));
1479 * Create a table.
1480 * @param tableName
1481 * @param family
1482 * @return A Table instance for the created table.
1483 * @throws IOException
1485 public Table createTable(TableName tableName, byte[] family)
1486 throws IOException{
1487 return createTable(tableName, new byte[][]{family});
1491 * Create a table with multiple regions.
1492 * @param tableName
1493 * @param family
1494 * @param numRegions
1495 * @return A Table instance for the created table.
1496 * @throws IOException
1498 public Table createMultiRegionTable(TableName tableName, byte[] family, int numRegions)
1499 throws IOException {
1500 if (numRegions < 3) throw new IOException("Must create at least 3 regions");
1501 byte[] startKey = Bytes.toBytes("aaaaa");
1502 byte[] endKey = Bytes.toBytes("zzzzz");
1503 byte[][] splitKeys = Bytes.split(startKey, endKey, numRegions - 3);
1505 return createTable(tableName, new byte[][] { family }, splitKeys);
1509 * Create a table.
1510 * @param tableName
1511 * @param families
1512 * @return A Table instance for the created table.
1513 * @throws IOException
1515 public Table createTable(TableName tableName, byte[][] families)
1516 throws IOException {
1517 return createTable(tableName, families, (byte[][]) null);
1521 * Create a table with multiple regions.
1522 * @param tableName
1523 * @param families
1524 * @return A Table instance for the created table.
1525 * @throws IOException
1527 public Table createMultiRegionTable(TableName tableName, byte[][] families) throws IOException {
1528 return createTable(tableName, families, KEYS_FOR_HBA_CREATE_TABLE);
1532 * Create a table.
1533 * @param tableName
1534 * @param families
1535 * @param splitKeys
1536 * @return A Table instance for the created table.
1537 * @throws IOException
1539 public Table createTable(TableName tableName, byte[][] families, byte[][] splitKeys)
1540 throws IOException {
1541 return createTable(tableName, families, splitKeys, 1, new Configuration(getConfiguration()));
1545 * Create a table.
1546 * @param tableName the table name
1547 * @param families the families
1548 * @param splitKeys the splitkeys
1549 * @param replicaCount the region replica count
1550 * @return A Table instance for the created table.
1551 * @throws IOException throws IOException
1553 public Table createTable(TableName tableName, byte[][] families, byte[][] splitKeys,
1554 int replicaCount) throws IOException {
1555 return createTable(tableName, families, splitKeys, replicaCount,
1556 new Configuration(getConfiguration()));
1559 public Table createTable(TableName tableName, byte[][] families,
1560 int numVersions, byte[] startKey, byte[] endKey, int numRegions)
1561 throws IOException{
1562 HTableDescriptor desc = createTableDescriptor(tableName, families, numVersions);
1564 getAdmin().createTable(desc, startKey, endKey, numRegions);
1565 // HBaseAdmin only waits for regions to appear in hbase:meta we
1566 // should wait until they are assigned
1567 waitUntilAllRegionsAssigned(tableName);
1568 return getConnection().getTable(tableName);
1572 * Create a table.
1573 * @param htd
1574 * @param families
1575 * @param c Configuration to use
1576 * @return A Table instance for the created table.
1577 * @throws IOException
1579 public Table createTable(TableDescriptor htd, byte[][] families, Configuration c)
1580 throws IOException {
1581 return createTable(htd, families, null, c);
1585 * Create a table.
1586 * @param htd table descriptor
1587 * @param families array of column families
1588 * @param splitKeys array of split keys
1589 * @param c Configuration to use
1590 * @return A Table instance for the created table.
1591 * @throws IOException if getAdmin or createTable fails
1593 public Table createTable(TableDescriptor htd, byte[][] families, byte[][] splitKeys,
1594 Configuration c) throws IOException {
1595 // Disable blooms (they are on by default as of 0.95) but we disable them here because
1596 // tests have hard coded counts of what to expect in block cache, etc., and blooms being
1597 // on is interfering.
1598 return createTable(htd, families, splitKeys, BloomType.NONE, HConstants.DEFAULT_BLOCKSIZE, c);
1602 * Create a table.
1603 * @param htd table descriptor
1604 * @param families array of column families
1605 * @param splitKeys array of split keys
1606 * @param type Bloom type
1607 * @param blockSize block size
1608 * @param c Configuration to use
1609 * @return A Table instance for the created table.
1610 * @throws IOException if getAdmin or createTable fails
1613 public Table createTable(TableDescriptor htd, byte[][] families, byte[][] splitKeys,
1614 BloomType type, int blockSize, Configuration c) throws IOException {
1615 TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(htd);
1616 for (byte[] family : families) {
1617 ColumnFamilyDescriptorBuilder cfdb = ColumnFamilyDescriptorBuilder.newBuilder(family)
1618 .setBloomFilterType(type)
1619 .setBlocksize(blockSize);
1620 if (isNewVersionBehaviorEnabled()) {
1621 cfdb.setNewVersionBehavior(true);
1623 builder.setColumnFamily(cfdb.build());
1625 TableDescriptor td = builder.build();
1626 if (splitKeys != null) {
1627 getAdmin().createTable(td, splitKeys);
1628 } else {
1629 getAdmin().createTable(td);
1631 // HBaseAdmin only waits for regions to appear in hbase:meta
1632 // we should wait until they are assigned
1633 waitUntilAllRegionsAssigned(td.getTableName());
1634 return getConnection().getTable(td.getTableName());
1638 * Create a table.
1639 * @param htd table descriptor
1640 * @param splitRows array of split keys
1641 * @return A Table instance for the created table.
1642 * @throws IOException
1644 public Table createTable(TableDescriptor htd, byte[][] splitRows)
1645 throws IOException {
1646 TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(htd);
1647 if (isNewVersionBehaviorEnabled()) {
1648 for (ColumnFamilyDescriptor family : htd.getColumnFamilies()) {
1649 builder.setColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(family)
1650 .setNewVersionBehavior(true).build());
1653 if (splitRows != null) {
1654 getAdmin().createTable(builder.build(), splitRows);
1655 } else {
1656 getAdmin().createTable(builder.build());
1658 // HBaseAdmin only waits for regions to appear in hbase:meta
1659 // we should wait until they are assigned
1660 waitUntilAllRegionsAssigned(htd.getTableName());
1661 return getConnection().getTable(htd.getTableName());
1665 * Create a table.
1666 * @param tableName the table name
1667 * @param families the families
1668 * @param splitKeys the split keys
1669 * @param replicaCount the replica count
1670 * @param c Configuration to use
1671 * @return A Table instance for the created table.
1672 * @throws IOException
1674 public Table createTable(TableName tableName, byte[][] families, byte[][] splitKeys,
1675 int replicaCount, final Configuration c) throws IOException {
1676 HTableDescriptor htd = new HTableDescriptor(tableName);
1677 htd.setRegionReplication(replicaCount);
1678 return createTable(htd, families, splitKeys, c);
1682 * Create a table.
1683 * @param tableName
1684 * @param family
1685 * @param numVersions
1686 * @return A Table instance for the created table.
1687 * @throws IOException
1689 public Table createTable(TableName tableName, byte[] family, int numVersions)
1690 throws IOException {
1691 return createTable(tableName, new byte[][]{family}, numVersions);
1695 * Create a table.
1696 * @param tableName
1697 * @param families
1698 * @param numVersions
1699 * @return A Table instance for the created table.
1700 * @throws IOException
1702 public Table createTable(TableName tableName, byte[][] families, int numVersions)
1703 throws IOException {
1704 return createTable(tableName, families, numVersions, (byte[][]) null);
1708 * Create a table.
1709 * @param tableName
1710 * @param families
1711 * @param numVersions
1712 * @param splitKeys
1713 * @return A Table instance for the created table.
1714 * @throws IOException
1716 public Table createTable(TableName tableName, byte[][] families, int numVersions,
1717 byte[][] splitKeys) throws IOException {
1718 HTableDescriptor desc = new HTableDescriptor(tableName);
1719 for (byte[] family : families) {
1720 HColumnDescriptor hcd = new HColumnDescriptor(family).setMaxVersions(numVersions);
1721 if (isNewVersionBehaviorEnabled()) {
1722 hcd.setNewVersionBehavior(true);
1724 desc.addFamily(hcd);
1726 if (splitKeys != null) {
1727 getAdmin().createTable(desc, splitKeys);
1728 } else {
1729 getAdmin().createTable(desc);
1731 // HBaseAdmin only waits for regions to appear in hbase:meta we should wait until they are
1732 // assigned
1733 waitUntilAllRegionsAssigned(tableName);
1734 return getConnection().getTable(tableName);
1738 * Create a table with multiple regions.
1739 * @param tableName
1740 * @param families
1741 * @param numVersions
1742 * @return A Table instance for the created table.
1743 * @throws IOException
1745 public Table createMultiRegionTable(TableName tableName, byte[][] families, int numVersions)
1746 throws IOException {
1747 return createTable(tableName, families, numVersions, KEYS_FOR_HBA_CREATE_TABLE);
1751 * Create a table.
1752 * @param tableName
1753 * @param families
1754 * @param numVersions
1755 * @param blockSize
1756 * @return A Table instance for the created table.
1757 * @throws IOException
1759 public Table createTable(TableName tableName, byte[][] families,
1760 int numVersions, int blockSize) throws IOException {
1761 HTableDescriptor desc = new HTableDescriptor(tableName);
1762 for (byte[] family : families) {
1763 HColumnDescriptor hcd = new HColumnDescriptor(family)
1764 .setMaxVersions(numVersions)
1765 .setBlocksize(blockSize);
1766 if (isNewVersionBehaviorEnabled()) {
1767 hcd.setNewVersionBehavior(true);
1769 desc.addFamily(hcd);
1771 getAdmin().createTable(desc);
1772 // HBaseAdmin only waits for regions to appear in hbase:meta we should wait until they are
1773 // assigned
1774 waitUntilAllRegionsAssigned(tableName);
1775 return getConnection().getTable(tableName);
1778 public Table createTable(TableName tableName, byte[][] families,
1779 int numVersions, int blockSize, String cpName) throws IOException {
1780 HTableDescriptor desc = new HTableDescriptor(tableName);
1781 for (byte[] family : families) {
1782 HColumnDescriptor hcd = new HColumnDescriptor(family)
1783 .setMaxVersions(numVersions)
1784 .setBlocksize(blockSize);
1785 if (isNewVersionBehaviorEnabled()) {
1786 hcd.setNewVersionBehavior(true);
1788 desc.addFamily(hcd);
1790 if(cpName != null) {
1791 desc.addCoprocessor(cpName);
1793 getAdmin().createTable(desc);
1794 // HBaseAdmin only waits for regions to appear in hbase:meta we should wait until they are
1795 // assigned
1796 waitUntilAllRegionsAssigned(tableName);
1797 return getConnection().getTable(tableName);
1801 * Create a table.
1802 * @param tableName
1803 * @param families
1804 * @param numVersions
1805 * @return A Table instance for the created table.
1806 * @throws IOException
1808 public Table createTable(TableName tableName, byte[][] families,
1809 int[] numVersions)
1810 throws IOException {
1811 HTableDescriptor desc = new HTableDescriptor(tableName);
1812 int i = 0;
1813 for (byte[] family : families) {
1814 HColumnDescriptor hcd = new HColumnDescriptor(family)
1815 .setMaxVersions(numVersions[i]);
1816 if (isNewVersionBehaviorEnabled()) {
1817 hcd.setNewVersionBehavior(true);
1819 desc.addFamily(hcd);
1820 i++;
1822 getAdmin().createTable(desc);
1823 // HBaseAdmin only waits for regions to appear in hbase:meta we should wait until they are
1824 // assigned
1825 waitUntilAllRegionsAssigned(tableName);
1826 return getConnection().getTable(tableName);
1830 * Create a table.
1831 * @param tableName
1832 * @param family
1833 * @param splitRows
1834 * @return A Table instance for the created table.
1835 * @throws IOException
1837 public Table createTable(TableName tableName, byte[] family, byte[][] splitRows)
1838 throws IOException {
1839 HTableDescriptor desc = new HTableDescriptor(tableName);
1840 HColumnDescriptor hcd = new HColumnDescriptor(family);
1841 if (isNewVersionBehaviorEnabled()) {
1842 hcd.setNewVersionBehavior(true);
1844 desc.addFamily(hcd);
1845 getAdmin().createTable(desc, splitRows);
1846 // HBaseAdmin only waits for regions to appear in hbase:meta we should wait until they are
1847 // assigned
1848 waitUntilAllRegionsAssigned(tableName);
1849 return getConnection().getTable(tableName);
1853 * Create a table with multiple regions.
1854 * @param tableName
1855 * @param family
1856 * @return A Table instance for the created table.
1857 * @throws IOException
1859 public Table createMultiRegionTable(TableName tableName, byte[] family) throws IOException {
1860 return createTable(tableName, family, KEYS_FOR_HBA_CREATE_TABLE);
1864 * Modify a table, synchronous.
1865 * @deprecated since 3.0.0 and will be removed in 4.0.0. Just use
1866 * {@link Admin#modifyTable(TableDescriptor)} directly as it is synchronous now.
1867 * @see Admin#modifyTable(TableDescriptor)
1868 * @see <a href="https://issues.apache.org/jira/browse/HBASE-22002">HBASE-22002</a>
1870 @Deprecated
1871 public static void modifyTableSync(Admin admin, TableDescriptor desc)
1872 throws IOException, InterruptedException {
1873 admin.modifyTable(desc);
1877 * Set the number of Region replicas.
1879 public static void setReplicas(Admin admin, TableName table, int replicaCount)
1880 throws IOException, InterruptedException {
1881 admin.disableTable(table);
1882 HTableDescriptor desc = new HTableDescriptor(admin.getDescriptor(table));
1883 desc.setRegionReplication(replicaCount);
1884 admin.modifyTable(desc);
1885 admin.enableTable(table);
1889 * Drop an existing table
1890 * @param tableName existing table
1892 public void deleteTable(TableName tableName) throws IOException {
1893 try {
1894 getAdmin().disableTable(tableName);
1895 } catch (TableNotEnabledException e) {
1896 LOG.debug("Table: " + tableName + " already disabled, so just deleting it.");
1898 getAdmin().deleteTable(tableName);
1902 * Drop an existing table
1903 * @param tableName existing table
1905 public void deleteTableIfAny(TableName tableName) throws IOException {
1906 try {
1907 deleteTable(tableName);
1908 } catch (TableNotFoundException e) {
1909 // ignore
1913 // ==========================================================================
1914 // Canned table and table descriptor creation
1915 // TODO replace HBaseTestCase
1917 public final static byte [] fam1 = Bytes.toBytes("colfamily11");
1918 public final static byte [] fam2 = Bytes.toBytes("colfamily21");
1919 public final static byte [] fam3 = Bytes.toBytes("colfamily31");
1920 public static final byte[][] COLUMNS = {fam1, fam2, fam3};
1921 private static final int MAXVERSIONS = 3;
1923 public static final char FIRST_CHAR = 'a';
1924 public static final char LAST_CHAR = 'z';
1925 public static final byte [] START_KEY_BYTES = {FIRST_CHAR, FIRST_CHAR, FIRST_CHAR};
1926 public static final String START_KEY = new String(START_KEY_BYTES, HConstants.UTF8_CHARSET);
1929 * @deprecated since 2.0.0 and will be removed in 3.0.0. Use
1930 * {@link #createTableDescriptor(TableName, int, int, int, KeepDeletedCells)} instead.
1931 * @see #createTableDescriptor(TableName, int, int, int, KeepDeletedCells)
1932 * @see <a href="https://issues.apache.org/jira/browse/HBASE-13893">HBASE-13893</a>
1934 @Deprecated
1935 public HTableDescriptor createTableDescriptor(final String name,
1936 final int minVersions, final int versions, final int ttl, KeepDeletedCells keepDeleted) {
1937 return this.createTableDescriptor(TableName.valueOf(name), minVersions, versions, ttl,
1938 keepDeleted);
1942 * Create a table of name <code>name</code>.
1943 * @param name Name to give table.
1944 * @return Column descriptor.
1945 * @deprecated since 2.0.0 and will be removed in 3.0.0. Use
1946 * {@link #createTableDescriptor(TableName, int, int, int, KeepDeletedCells)} instead.
1947 * @see #createTableDescriptor(TableName, int, int, int, KeepDeletedCells)
1948 * @see <a href="https://issues.apache.org/jira/browse/HBASE-13893">HBASE-13893</a>
1950 @Deprecated
1951 public HTableDescriptor createTableDescriptor(final String name) {
1952 return createTableDescriptor(TableName.valueOf(name), HColumnDescriptor.DEFAULT_MIN_VERSIONS,
1953 MAXVERSIONS, HConstants.FOREVER, HColumnDescriptor.DEFAULT_KEEP_DELETED);
1956 public HTableDescriptor createTableDescriptor(final TableName name,
1957 final int minVersions, final int versions, final int ttl, KeepDeletedCells keepDeleted) {
1958 HTableDescriptor htd = new HTableDescriptor(name);
1959 for (byte[] cfName : new byte[][]{ fam1, fam2, fam3 }) {
1960 HColumnDescriptor hcd = new HColumnDescriptor(cfName)
1961 .setMinVersions(minVersions)
1962 .setMaxVersions(versions)
1963 .setKeepDeletedCells(keepDeleted)
1964 .setBlockCacheEnabled(false)
1965 .setTimeToLive(ttl);
1966 if (isNewVersionBehaviorEnabled()) {
1967 hcd.setNewVersionBehavior(true);
1969 htd.addFamily(hcd);
1971 return htd;
1975 * Create a table of name <code>name</code>.
1976 * @param name Name to give table.
1977 * @return Column descriptor.
1979 public HTableDescriptor createTableDescriptor(final TableName name) {
1980 return createTableDescriptor(name, HColumnDescriptor.DEFAULT_MIN_VERSIONS,
1981 MAXVERSIONS, HConstants.FOREVER, HColumnDescriptor.DEFAULT_KEEP_DELETED);
1984 public HTableDescriptor createTableDescriptor(final TableName tableName,
1985 byte[] family) {
1986 return createTableDescriptor(tableName, new byte[][] {family}, 1);
1989 public HTableDescriptor createTableDescriptor(final TableName tableName,
1990 byte[][] families, int maxVersions) {
1991 HTableDescriptor desc = new HTableDescriptor(tableName);
1992 for (byte[] family : families) {
1993 HColumnDescriptor hcd = new HColumnDescriptor(family)
1994 .setMaxVersions(maxVersions);
1995 if (isNewVersionBehaviorEnabled()) {
1996 hcd.setNewVersionBehavior(true);
1998 desc.addFamily(hcd);
2000 return desc;
2004 * Create an HRegion that writes to the local tmp dirs
2005 * @param desc a table descriptor indicating which table the region belongs to
2006 * @param startKey the start boundary of the region
2007 * @param endKey the end boundary of the region
2008 * @return a region that writes to local dir for testing
2009 * @throws IOException
2011 public HRegion createLocalHRegion(TableDescriptor desc, byte [] startKey,
2012 byte [] endKey)
2013 throws IOException {
2014 HRegionInfo hri = new HRegionInfo(desc.getTableName(), startKey, endKey);
2015 return createLocalHRegion(hri, desc);
2019 * Create an HRegion that writes to the local tmp dirs. Creates the WAL for you. Be sure to call
2020 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} when you're finished with it.
2022 public HRegion createLocalHRegion(RegionInfo info, TableDescriptor desc) throws IOException {
2023 return createRegionAndWAL(info, getDataTestDir(), getConfiguration(), desc);
2027 * Create an HRegion that writes to the local tmp dirs with specified wal
2028 * @param info regioninfo
2029 * @param desc table descriptor
2030 * @param wal wal for this region.
2031 * @return created hregion
2032 * @throws IOException
2034 public HRegion createLocalHRegion(RegionInfo info, TableDescriptor desc, WAL wal)
2035 throws IOException {
2036 return HRegion.createHRegion(info, getDataTestDir(), getConfiguration(), desc, wal);
2040 * @param tableName the name of the table
2041 * @param startKey the start key of the region
2042 * @param stopKey the stop key of the region
2043 * @param callingMethod the name of the calling method probably a test method
2044 * @param conf the configuration to use
2045 * @param isReadOnly {@code true} if the table is read only, {@code false} otherwise
2046 * @param families the column families to use
2047 * @throws IOException if an IO problem is encountered
2048 * @return A region on which you must call {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)}
2049 * when done.
2050 * @deprecated since 2.0.0 and will be removed in 3.0.0. Use
2051 * {@link #createLocalHRegion(TableName, byte[], byte[], boolean, Durability, WAL, byte[]...)}
2052 * instead.
2053 * @see #createLocalHRegion(TableName, byte[], byte[], boolean, Durability, WAL, byte[]...)
2054 * @see <a href="https://issues.apache.org/jira/browse/HBASE-13893">HBASE-13893</a>
2056 @Deprecated
2057 public HRegion createLocalHRegion(byte[] tableName, byte[] startKey, byte[] stopKey,
2058 String callingMethod, Configuration conf, boolean isReadOnly, Durability durability,
2059 WAL wal, byte[]... families) throws IOException {
2060 return this
2061 .createLocalHRegion(TableName.valueOf(tableName), startKey, stopKey, isReadOnly, durability,
2062 wal, families);
2066 * @param tableName
2067 * @param startKey
2068 * @param stopKey
2069 * @param isReadOnly
2070 * @param families
2071 * @return A region on which you must call
2072 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} when done.
2073 * @throws IOException
2075 public HRegion createLocalHRegion(TableName tableName, byte[] startKey, byte[] stopKey,
2076 boolean isReadOnly, Durability durability, WAL wal, byte[]... families) throws IOException {
2077 return createLocalHRegionWithInMemoryFlags(tableName,startKey, stopKey, isReadOnly,
2078 durability, wal, null, families);
2081 public HRegion createLocalHRegionWithInMemoryFlags(TableName tableName, byte[] startKey,
2082 byte[] stopKey,
2083 boolean isReadOnly, Durability durability, WAL wal, boolean[] compactedMemStore,
2084 byte[]... families)
2085 throws IOException {
2086 HTableDescriptor htd = new HTableDescriptor(tableName);
2087 htd.setReadOnly(isReadOnly);
2088 int i=0;
2089 for (byte[] family : families) {
2090 HColumnDescriptor hcd = new HColumnDescriptor(family);
2091 if(compactedMemStore != null && i < compactedMemStore.length) {
2092 hcd.setInMemoryCompaction(MemoryCompactionPolicy.BASIC);
2093 } else {
2094 hcd.setInMemoryCompaction(MemoryCompactionPolicy.NONE);
2097 i++;
2098 // Set default to be three versions.
2099 hcd.setMaxVersions(Integer.MAX_VALUE);
2100 htd.addFamily(hcd);
2102 htd.setDurability(durability);
2103 HRegionInfo info = new HRegionInfo(htd.getTableName(), startKey, stopKey, false);
2104 return createLocalHRegion(info, htd, wal);
2108 // ==========================================================================
2111 * Provide an existing table name to truncate.
2112 * Scans the table and issues a delete for each row read.
2113 * @param tableName existing table
2114 * @return HTable to that new table
2115 * @throws IOException
2117 public Table deleteTableData(TableName tableName) throws IOException {
2118 Table table = getConnection().getTable(tableName);
2119 Scan scan = new Scan();
2120 ResultScanner resScan = table.getScanner(scan);
2121 for(Result res : resScan) {
2122 Delete del = new Delete(res.getRow());
2123 table.delete(del);
2125 resScan = table.getScanner(scan);
2126 resScan.close();
2127 return table;
2131 * Truncate a table using the admin command.
2132 * Effectively disables, deletes, and recreates the table.
2133 * @param tableName table which must exist.
2134 * @param preserveRegions keep the existing split points
2135 * @return HTable for the new table
2137 public Table truncateTable(final TableName tableName, final boolean preserveRegions) throws
2138 IOException {
2139 Admin admin = getAdmin();
2140 if (!admin.isTableDisabled(tableName)) {
2141 admin.disableTable(tableName);
2143 admin.truncateTable(tableName, preserveRegions);
2144 return getConnection().getTable(tableName);
2148 * Truncate a table using the admin command.
2149 * Effectively disables, deletes, and recreates the table.
2150 * For previous behavior of issuing row deletes, see
2151 * deleteTableData.
2152 * Expressly does not preserve regions of existing table.
2153 * @param tableName table which must exist.
2154 * @return HTable for the new table
2156 public Table truncateTable(final TableName tableName) throws IOException {
2157 return truncateTable(tableName, false);
2161 * Load table with rows from 'aaa' to 'zzz'.
2162 * @param t Table
2163 * @param f Family
2164 * @return Count of rows loaded.
2165 * @throws IOException
2167 public int loadTable(final Table t, final byte[] f) throws IOException {
2168 return loadTable(t, new byte[][] {f});
2172 * Load table with rows from 'aaa' to 'zzz'.
2173 * @param t Table
2174 * @param f Family
2175 * @return Count of rows loaded.
2176 * @throws IOException
2178 public int loadTable(final Table t, final byte[] f, boolean writeToWAL) throws IOException {
2179 return loadTable(t, new byte[][] {f}, null, writeToWAL);
2183 * Load table of multiple column families with rows from 'aaa' to 'zzz'.
2184 * @param t Table
2185 * @param f Array of Families to load
2186 * @return Count of rows loaded.
2187 * @throws IOException
2189 public int loadTable(final Table t, final byte[][] f) throws IOException {
2190 return loadTable(t, f, null);
2194 * Load table of multiple column families with rows from 'aaa' to 'zzz'.
2195 * @param t Table
2196 * @param f Array of Families to load
2197 * @param value the values of the cells. If null is passed, the row key is used as value
2198 * @return Count of rows loaded.
2199 * @throws IOException
2201 public int loadTable(final Table t, final byte[][] f, byte[] value) throws IOException {
2202 return loadTable(t, f, value, true);
2206 * Load table of multiple column families with rows from 'aaa' to 'zzz'.
2207 * @param t Table
2208 * @param f Array of Families to load
2209 * @param value the values of the cells. If null is passed, the row key is used as value
2210 * @return Count of rows loaded.
2211 * @throws IOException
2213 public int loadTable(final Table t, final byte[][] f, byte[] value,
2214 boolean writeToWAL) throws IOException {
2215 List<Put> puts = new ArrayList<>();
2216 for (byte[] row : HBaseTestingUtility.ROWS) {
2217 Put put = new Put(row);
2218 put.setDurability(writeToWAL ? Durability.USE_DEFAULT : Durability.SKIP_WAL);
2219 for (int i = 0; i < f.length; i++) {
2220 byte[] value1 = value != null ? value : row;
2221 put.addColumn(f[i], f[i], value1);
2223 puts.add(put);
2225 t.put(puts);
2226 return puts.size();
2229 /** A tracker for tracking and validating table rows
2230 * generated with {@link HBaseTestingUtility#loadTable(Table, byte[])}
2232 public static class SeenRowTracker {
2233 int dim = 'z' - 'a' + 1;
2234 int[][][] seenRows = new int[dim][dim][dim]; //count of how many times the row is seen
2235 byte[] startRow;
2236 byte[] stopRow;
2238 public SeenRowTracker(byte[] startRow, byte[] stopRow) {
2239 this.startRow = startRow;
2240 this.stopRow = stopRow;
2243 void reset() {
2244 for (byte[] row : ROWS) {
2245 seenRows[i(row[0])][i(row[1])][i(row[2])] = 0;
2249 int i(byte b) {
2250 return b - 'a';
2253 public void addRow(byte[] row) {
2254 seenRows[i(row[0])][i(row[1])][i(row[2])]++;
2257 /** Validate that all the rows between startRow and stopRow are seen exactly once, and
2258 * all other rows none
2260 public void validate() {
2261 for (byte b1 = 'a'; b1 <= 'z'; b1++) {
2262 for (byte b2 = 'a'; b2 <= 'z'; b2++) {
2263 for (byte b3 = 'a'; b3 <= 'z'; b3++) {
2264 int count = seenRows[i(b1)][i(b2)][i(b3)];
2265 int expectedCount = 0;
2266 if (Bytes.compareTo(new byte[] {b1,b2,b3}, startRow) >= 0
2267 && Bytes.compareTo(new byte[] {b1,b2,b3}, stopRow) < 0) {
2268 expectedCount = 1;
2270 if (count != expectedCount) {
2271 String row = new String(new byte[] {b1,b2,b3}, StandardCharsets.UTF_8);
2272 throw new RuntimeException("Row:" + row + " has a seen count of " + count + " " +
2273 "instead of " + expectedCount);
2281 public int loadRegion(final HRegion r, final byte[] f) throws IOException {
2282 return loadRegion(r, f, false);
2285 public int loadRegion(final Region r, final byte[] f) throws IOException {
2286 return loadRegion((HRegion)r, f);
2290 * Load region with rows from 'aaa' to 'zzz'.
2291 * @param r Region
2292 * @param f Family
2293 * @param flush flush the cache if true
2294 * @return Count of rows loaded.
2295 * @throws IOException
2297 public int loadRegion(final HRegion r, final byte[] f, final boolean flush)
2298 throws IOException {
2299 byte[] k = new byte[3];
2300 int rowCount = 0;
2301 for (byte b1 = 'a'; b1 <= 'z'; b1++) {
2302 for (byte b2 = 'a'; b2 <= 'z'; b2++) {
2303 for (byte b3 = 'a'; b3 <= 'z'; b3++) {
2304 k[0] = b1;
2305 k[1] = b2;
2306 k[2] = b3;
2307 Put put = new Put(k);
2308 put.setDurability(Durability.SKIP_WAL);
2309 put.addColumn(f, null, k);
2310 if (r.getWAL() == null) {
2311 put.setDurability(Durability.SKIP_WAL);
2313 int preRowCount = rowCount;
2314 int pause = 10;
2315 int maxPause = 1000;
2316 while (rowCount == preRowCount) {
2317 try {
2318 r.put(put);
2319 rowCount++;
2320 } catch (RegionTooBusyException e) {
2321 pause = (pause * 2 >= maxPause) ? maxPause : pause * 2;
2322 Threads.sleep(pause);
2327 if (flush) {
2328 r.flush(true);
2331 return rowCount;
2334 public void loadNumericRows(final Table t, final byte[] f, int startRow, int endRow)
2335 throws IOException {
2336 for (int i = startRow; i < endRow; i++) {
2337 byte[] data = Bytes.toBytes(String.valueOf(i));
2338 Put put = new Put(data);
2339 put.addColumn(f, null, data);
2340 t.put(put);
2344 public void loadRandomRows(final Table t, final byte[] f, int rowSize, int totalRows)
2345 throws IOException {
2346 Random r = new Random();
2347 byte[] row = new byte[rowSize];
2348 for (int i = 0; i < totalRows; i++) {
2349 r.nextBytes(row);
2350 Put put = new Put(row);
2351 put.addColumn(f, new byte[]{0}, new byte[]{0});
2352 t.put(put);
2356 public void verifyNumericRows(Table table, final byte[] f, int startRow, int endRow,
2357 int replicaId)
2358 throws IOException {
2359 for (int i = startRow; i < endRow; i++) {
2360 String failMsg = "Failed verification of row :" + i;
2361 byte[] data = Bytes.toBytes(String.valueOf(i));
2362 Get get = new Get(data);
2363 get.setReplicaId(replicaId);
2364 get.setConsistency(Consistency.TIMELINE);
2365 Result result = table.get(get);
2366 assertTrue(failMsg, result.containsColumn(f, null));
2367 assertEquals(failMsg, 1, result.getColumnCells(f, null).size());
2368 Cell cell = result.getColumnLatestCell(f, null);
2369 assertTrue(failMsg,
2370 Bytes.equals(data, 0, data.length, cell.getValueArray(), cell.getValueOffset(),
2371 cell.getValueLength()));
2375 public void verifyNumericRows(Region region, final byte[] f, int startRow, int endRow)
2376 throws IOException {
2377 verifyNumericRows((HRegion)region, f, startRow, endRow);
2380 public void verifyNumericRows(HRegion region, final byte[] f, int startRow, int endRow)
2381 throws IOException {
2382 verifyNumericRows(region, f, startRow, endRow, true);
2385 public void verifyNumericRows(Region region, final byte[] f, int startRow, int endRow,
2386 final boolean present) throws IOException {
2387 verifyNumericRows((HRegion)region, f, startRow, endRow, present);
2390 public void verifyNumericRows(HRegion region, final byte[] f, int startRow, int endRow,
2391 final boolean present) throws IOException {
2392 for (int i = startRow; i < endRow; i++) {
2393 String failMsg = "Failed verification of row :" + i;
2394 byte[] data = Bytes.toBytes(String.valueOf(i));
2395 Result result = region.get(new Get(data));
2397 boolean hasResult = result != null && !result.isEmpty();
2398 assertEquals(failMsg + result, present, hasResult);
2399 if (!present) continue;
2401 assertTrue(failMsg, result.containsColumn(f, null));
2402 assertEquals(failMsg, 1, result.getColumnCells(f, null).size());
2403 Cell cell = result.getColumnLatestCell(f, null);
2404 assertTrue(failMsg,
2405 Bytes.equals(data, 0, data.length, cell.getValueArray(), cell.getValueOffset(),
2406 cell.getValueLength()));
2410 public void deleteNumericRows(final Table t, final byte[] f, int startRow, int endRow)
2411 throws IOException {
2412 for (int i = startRow; i < endRow; i++) {
2413 byte[] data = Bytes.toBytes(String.valueOf(i));
2414 Delete delete = new Delete(data);
2415 delete.addFamily(f);
2416 t.delete(delete);
2421 * Return the number of rows in the given table.
2422 * @param table to count rows
2423 * @return count of rows
2425 public static int countRows(final Table table) throws IOException {
2426 return countRows(table, new Scan());
2429 public static int countRows(final Table table, final Scan scan) throws IOException {
2430 try (ResultScanner results = table.getScanner(scan)) {
2431 int count = 0;
2432 while (results.next() != null) {
2433 count++;
2435 return count;
2439 public int countRows(final Table table, final byte[]... families) throws IOException {
2440 Scan scan = new Scan();
2441 for (byte[] family: families) {
2442 scan.addFamily(family);
2444 return countRows(table, scan);
2448 * Return the number of rows in the given table.
2450 public int countRows(final TableName tableName) throws IOException {
2451 Table table = getConnection().getTable(tableName);
2452 try {
2453 return countRows(table);
2454 } finally {
2455 table.close();
2459 public int countRows(final Region region) throws IOException {
2460 return countRows(region, new Scan());
2463 public int countRows(final Region region, final Scan scan) throws IOException {
2464 InternalScanner scanner = region.getScanner(scan);
2465 try {
2466 return countRows(scanner);
2467 } finally {
2468 scanner.close();
2472 public int countRows(final InternalScanner scanner) throws IOException {
2473 int scannedCount = 0;
2474 List<Cell> results = new ArrayList<>();
2475 boolean hasMore = true;
2476 while (hasMore) {
2477 hasMore = scanner.next(results);
2478 scannedCount += results.size();
2479 results.clear();
2481 return scannedCount;
2485 * Return an md5 digest of the entire contents of a table.
2487 public String checksumRows(final Table table) throws Exception {
2489 Scan scan = new Scan();
2490 ResultScanner results = table.getScanner(scan);
2491 MessageDigest digest = MessageDigest.getInstance("MD5");
2492 for (Result res : results) {
2493 digest.update(res.getRow());
2495 results.close();
2496 return digest.toString();
2499 /** All the row values for the data loaded by {@link #loadTable(Table, byte[])} */
2500 public static final byte[][] ROWS = new byte[(int) Math.pow('z' - 'a' + 1, 3)][3]; // ~52KB
2501 static {
2502 int i = 0;
2503 for (byte b1 = 'a'; b1 <= 'z'; b1++) {
2504 for (byte b2 = 'a'; b2 <= 'z'; b2++) {
2505 for (byte b3 = 'a'; b3 <= 'z'; b3++) {
2506 ROWS[i][0] = b1;
2507 ROWS[i][1] = b2;
2508 ROWS[i][2] = b3;
2509 i++;
2515 public static final byte[][] KEYS = {
2516 HConstants.EMPTY_BYTE_ARRAY, Bytes.toBytes("bbb"),
2517 Bytes.toBytes("ccc"), Bytes.toBytes("ddd"), Bytes.toBytes("eee"),
2518 Bytes.toBytes("fff"), Bytes.toBytes("ggg"), Bytes.toBytes("hhh"),
2519 Bytes.toBytes("iii"), Bytes.toBytes("jjj"), Bytes.toBytes("kkk"),
2520 Bytes.toBytes("lll"), Bytes.toBytes("mmm"), Bytes.toBytes("nnn"),
2521 Bytes.toBytes("ooo"), Bytes.toBytes("ppp"), Bytes.toBytes("qqq"),
2522 Bytes.toBytes("rrr"), Bytes.toBytes("sss"), Bytes.toBytes("ttt"),
2523 Bytes.toBytes("uuu"), Bytes.toBytes("vvv"), Bytes.toBytes("www"),
2524 Bytes.toBytes("xxx"), Bytes.toBytes("yyy")
2527 public static final byte[][] KEYS_FOR_HBA_CREATE_TABLE = {
2528 Bytes.toBytes("bbb"),
2529 Bytes.toBytes("ccc"), Bytes.toBytes("ddd"), Bytes.toBytes("eee"),
2530 Bytes.toBytes("fff"), Bytes.toBytes("ggg"), Bytes.toBytes("hhh"),
2531 Bytes.toBytes("iii"), Bytes.toBytes("jjj"), Bytes.toBytes("kkk"),
2532 Bytes.toBytes("lll"), Bytes.toBytes("mmm"), Bytes.toBytes("nnn"),
2533 Bytes.toBytes("ooo"), Bytes.toBytes("ppp"), Bytes.toBytes("qqq"),
2534 Bytes.toBytes("rrr"), Bytes.toBytes("sss"), Bytes.toBytes("ttt"),
2535 Bytes.toBytes("uuu"), Bytes.toBytes("vvv"), Bytes.toBytes("www"),
2536 Bytes.toBytes("xxx"), Bytes.toBytes("yyy"), Bytes.toBytes("zzz")
2540 * Create rows in hbase:meta for regions of the specified table with the specified
2541 * start keys. The first startKey should be a 0 length byte array if you
2542 * want to form a proper range of regions.
2543 * @param conf
2544 * @param htd
2545 * @param startKeys
2546 * @return list of region info for regions added to meta
2547 * @throws IOException
2549 public List<RegionInfo> createMultiRegionsInMeta(final Configuration conf,
2550 final TableDescriptor htd, byte [][] startKeys)
2551 throws IOException {
2552 Table meta = getConnection().getTable(TableName.META_TABLE_NAME);
2553 Arrays.sort(startKeys, Bytes.BYTES_COMPARATOR);
2554 List<RegionInfo> newRegions = new ArrayList<>(startKeys.length);
2555 MetaTableAccessor
2556 .updateTableState(getConnection(), htd.getTableName(), TableState.State.ENABLED);
2557 // add custom ones
2558 for (int i = 0; i < startKeys.length; i++) {
2559 int j = (i + 1) % startKeys.length;
2560 RegionInfo hri = RegionInfoBuilder.newBuilder(htd.getTableName())
2561 .setStartKey(startKeys[i])
2562 .setEndKey(startKeys[j])
2563 .build();
2564 MetaTableAccessor.addRegionToMeta(getConnection(), hri);
2565 newRegions.add(hri);
2568 meta.close();
2569 return newRegions;
2573 * Create an unmanaged WAL. Be sure to close it when you're through.
2575 public static WAL createWal(final Configuration conf, final Path rootDir, final RegionInfo hri)
2576 throws IOException {
2577 // The WAL subsystem will use the default rootDir rather than the passed in rootDir
2578 // unless I pass along via the conf.
2579 Configuration confForWAL = new Configuration(conf);
2580 confForWAL.set(HConstants.HBASE_DIR, rootDir.toString());
2581 return new WALFactory(confForWAL, "hregion-" + RandomStringUtils.randomNumeric(8)).getWAL(hri);
2586 * Create a region with it's own WAL. Be sure to call
2587 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} to clean up all resources.
2589 public static HRegion createRegionAndWAL(final RegionInfo info, final Path rootDir,
2590 final Configuration conf, final TableDescriptor htd) throws IOException {
2591 return createRegionAndWAL(info, rootDir, conf, htd, true);
2595 * Create a region with it's own WAL. Be sure to call
2596 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} to clean up all resources.
2598 public static HRegion createRegionAndWAL(final RegionInfo info, final Path rootDir,
2599 final Configuration conf, final TableDescriptor htd, BlockCache blockCache)
2600 throws IOException {
2601 HRegion region = createRegionAndWAL(info, rootDir, conf, htd, false);
2602 region.setBlockCache(blockCache);
2603 region.initialize();
2604 return region;
2607 * Create a region with it's own WAL. Be sure to call
2608 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} to clean up all resources.
2610 public static HRegion createRegionAndWAL(final RegionInfo info, final Path rootDir,
2611 final Configuration conf, final TableDescriptor htd, MobFileCache mobFileCache)
2612 throws IOException {
2613 HRegion region = createRegionAndWAL(info, rootDir, conf, htd, false);
2614 region.setMobFileCache(mobFileCache);
2615 region.initialize();
2616 return region;
2620 * Create a region with it's own WAL. Be sure to call
2621 * {@link HBaseTestingUtility#closeRegionAndWAL(HRegion)} to clean up all resources.
2623 public static HRegion createRegionAndWAL(final RegionInfo info, final Path rootDir,
2624 final Configuration conf, final TableDescriptor htd, boolean initialize)
2625 throws IOException {
2626 ChunkCreator.initialize(MemStoreLABImpl.CHUNK_SIZE_DEFAULT, false, 0, 0, 0, null);
2627 WAL wal = createWal(conf, rootDir, info);
2628 return HRegion.createHRegion(info, rootDir, conf, htd, wal, initialize);
2632 * Returns all rows from the hbase:meta table.
2634 * @throws IOException When reading the rows fails.
2636 public List<byte[]> getMetaTableRows() throws IOException {
2637 // TODO: Redo using MetaTableAccessor class
2638 Table t = getConnection().getTable(TableName.META_TABLE_NAME);
2639 List<byte[]> rows = new ArrayList<>();
2640 ResultScanner s = t.getScanner(new Scan());
2641 for (Result result : s) {
2642 LOG.info("getMetaTableRows: row -> " +
2643 Bytes.toStringBinary(result.getRow()));
2644 rows.add(result.getRow());
2646 s.close();
2647 t.close();
2648 return rows;
2652 * Returns all rows from the hbase:meta table for a given user table
2654 * @throws IOException When reading the rows fails.
2656 public List<byte[]> getMetaTableRows(TableName tableName) throws IOException {
2657 // TODO: Redo using MetaTableAccessor.
2658 Table t = getConnection().getTable(TableName.META_TABLE_NAME);
2659 List<byte[]> rows = new ArrayList<>();
2660 ResultScanner s = t.getScanner(new Scan());
2661 for (Result result : s) {
2662 RegionInfo info = MetaTableAccessor.getRegionInfo(result);
2663 if (info == null) {
2664 LOG.error("No region info for row " + Bytes.toString(result.getRow()));
2665 // TODO figure out what to do for this new hosed case.
2666 continue;
2669 if (info.getTable().equals(tableName)) {
2670 LOG.info("getMetaTableRows: row -> " +
2671 Bytes.toStringBinary(result.getRow()) + info);
2672 rows.add(result.getRow());
2675 s.close();
2676 t.close();
2677 return rows;
2681 * Returns all regions of the specified table
2683 * @param tableName the table name
2684 * @return all regions of the specified table
2685 * @throws IOException when getting the regions fails.
2687 private List<RegionInfo> getRegions(TableName tableName) throws IOException {
2688 try (Admin admin = getConnection().getAdmin()) {
2689 return admin.getRegions(tableName);
2694 * Find any other region server which is different from the one identified by parameter
2695 * @param rs
2696 * @return another region server
2698 public HRegionServer getOtherRegionServer(HRegionServer rs) {
2699 for (JVMClusterUtil.RegionServerThread rst :
2700 getMiniHBaseCluster().getRegionServerThreads()) {
2701 if (!(rst.getRegionServer() == rs)) {
2702 return rst.getRegionServer();
2705 return null;
2709 * Tool to get the reference to the region server object that holds the
2710 * region of the specified user table.
2711 * @param tableName user table to lookup in hbase:meta
2712 * @return region server that holds it, null if the row doesn't exist
2713 * @throws IOException
2714 * @throws InterruptedException
2716 public HRegionServer getRSForFirstRegionInTable(TableName tableName)
2717 throws IOException, InterruptedException {
2718 List<RegionInfo> regions = getRegions(tableName);
2719 if (regions == null || regions.isEmpty()) {
2720 return null;
2722 LOG.debug("Found " + regions.size() + " regions for table " +
2723 tableName);
2725 byte[] firstRegionName = regions.stream()
2726 .filter(r -> !r.isOffline())
2727 .map(RegionInfo::getRegionName)
2728 .findFirst()
2729 .orElseThrow(() -> new IOException("online regions not found in table " + tableName));
2731 LOG.debug("firstRegionName=" + Bytes.toString(firstRegionName));
2732 long pause = getConfiguration().getLong(HConstants.HBASE_CLIENT_PAUSE,
2733 HConstants.DEFAULT_HBASE_CLIENT_PAUSE);
2734 int numRetries = getConfiguration().getInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER,
2735 HConstants.DEFAULT_HBASE_CLIENT_RETRIES_NUMBER);
2736 RetryCounter retrier = new RetryCounter(numRetries+1, (int)pause, TimeUnit.MICROSECONDS);
2737 while(retrier.shouldRetry()) {
2738 int index = getMiniHBaseCluster().getServerWith(firstRegionName);
2739 if (index != -1) {
2740 return getMiniHBaseCluster().getRegionServerThreads().get(index).getRegionServer();
2742 // Came back -1. Region may not be online yet. Sleep a while.
2743 retrier.sleepUntilNextRetry();
2745 return null;
2749 * Starts a <code>MiniMRCluster</code> with a default number of
2750 * <code>TaskTracker</code>'s.
2752 * @throws IOException When starting the cluster fails.
2754 public MiniMRCluster startMiniMapReduceCluster() throws IOException {
2755 // Set a very high max-disk-utilization percentage to avoid the NodeManagers from failing.
2756 conf.setIfUnset(
2757 "yarn.nodemanager.disk-health-checker.max-disk-utilization-per-disk-percentage",
2758 "99.0");
2759 startMiniMapReduceCluster(2);
2760 return mrCluster;
2764 * Tasktracker has a bug where changing the hadoop.log.dir system property
2765 * will not change its internal static LOG_DIR variable.
2767 private void forceChangeTaskLogDir() {
2768 Field logDirField;
2769 try {
2770 logDirField = TaskLog.class.getDeclaredField("LOG_DIR");
2771 logDirField.setAccessible(true);
2773 Field modifiersField = Field.class.getDeclaredField("modifiers");
2774 modifiersField.setAccessible(true);
2775 modifiersField.setInt(logDirField, logDirField.getModifiers() & ~Modifier.FINAL);
2777 logDirField.set(null, new File(hadoopLogDir, "userlogs"));
2778 } catch (SecurityException e) {
2779 throw new RuntimeException(e);
2780 } catch (NoSuchFieldException e) {
2781 // TODO Auto-generated catch block
2782 throw new RuntimeException(e);
2783 } catch (IllegalArgumentException e) {
2784 throw new RuntimeException(e);
2785 } catch (IllegalAccessException e) {
2786 throw new RuntimeException(e);
2791 * Starts a <code>MiniMRCluster</code>. Call {@link #setFileSystemURI(String)} to use a different
2792 * filesystem.
2793 * @param servers The number of <code>TaskTracker</code>'s to start.
2794 * @throws IOException When starting the cluster fails.
2796 private void startMiniMapReduceCluster(final int servers) throws IOException {
2797 if (mrCluster != null) {
2798 throw new IllegalStateException("MiniMRCluster is already running");
2800 LOG.info("Starting mini mapreduce cluster...");
2801 setupClusterTestDir();
2802 createDirsAndSetProperties();
2804 forceChangeTaskLogDir();
2806 //// hadoop2 specific settings
2807 // Tests were failing because this process used 6GB of virtual memory and was getting killed.
2808 // we up the VM usable so that processes don't get killed.
2809 conf.setFloat("yarn.nodemanager.vmem-pmem-ratio", 8.0f);
2811 // Tests were failing due to MAPREDUCE-4880 / MAPREDUCE-4607 against hadoop 2.0.2-alpha and
2812 // this avoids the problem by disabling speculative task execution in tests.
2813 conf.setBoolean("mapreduce.map.speculative", false);
2814 conf.setBoolean("mapreduce.reduce.speculative", false);
2815 ////
2817 // Allow the user to override FS URI for this map-reduce cluster to use.
2818 mrCluster = new MiniMRCluster(servers,
2819 FS_URI != null ? FS_URI : FileSystem.get(conf).getUri().toString(), 1,
2820 null, null, new JobConf(this.conf));
2821 JobConf jobConf = MapreduceTestingShim.getJobConf(mrCluster);
2822 if (jobConf == null) {
2823 jobConf = mrCluster.createJobConf();
2826 jobConf.set("mapreduce.cluster.local.dir",
2827 conf.get("mapreduce.cluster.local.dir")); //Hadoop MiniMR overwrites this while it should not
2828 LOG.info("Mini mapreduce cluster started");
2830 // In hadoop2, YARN/MR2 starts a mini cluster with its own conf instance and updates settings.
2831 // Our HBase MR jobs need several of these settings in order to properly run. So we copy the
2832 // necessary config properties here. YARN-129 required adding a few properties.
2833 conf.set("mapreduce.jobtracker.address", jobConf.get("mapreduce.jobtracker.address"));
2834 // this for mrv2 support; mr1 ignores this
2835 conf.set("mapreduce.framework.name", "yarn");
2836 conf.setBoolean("yarn.is.minicluster", true);
2837 String rmAddress = jobConf.get("yarn.resourcemanager.address");
2838 if (rmAddress != null) {
2839 conf.set("yarn.resourcemanager.address", rmAddress);
2841 String historyAddress = jobConf.get("mapreduce.jobhistory.address");
2842 if (historyAddress != null) {
2843 conf.set("mapreduce.jobhistory.address", historyAddress);
2845 String schedulerAddress =
2846 jobConf.get("yarn.resourcemanager.scheduler.address");
2847 if (schedulerAddress != null) {
2848 conf.set("yarn.resourcemanager.scheduler.address", schedulerAddress);
2850 String mrJobHistoryWebappAddress =
2851 jobConf.get("mapreduce.jobhistory.webapp.address");
2852 if (mrJobHistoryWebappAddress != null) {
2853 conf.set("mapreduce.jobhistory.webapp.address", mrJobHistoryWebappAddress);
2855 String yarnRMWebappAddress =
2856 jobConf.get("yarn.resourcemanager.webapp.address");
2857 if (yarnRMWebappAddress != null) {
2858 conf.set("yarn.resourcemanager.webapp.address", yarnRMWebappAddress);
2863 * Stops the previously started <code>MiniMRCluster</code>.
2865 public void shutdownMiniMapReduceCluster() {
2866 if (mrCluster != null) {
2867 LOG.info("Stopping mini mapreduce cluster...");
2868 mrCluster.shutdown();
2869 mrCluster = null;
2870 LOG.info("Mini mapreduce cluster stopped");
2872 // Restore configuration to point to local jobtracker
2873 conf.set("mapreduce.jobtracker.address", "local");
2877 * Create a stubbed out RegionServerService, mainly for getting FS.
2879 public RegionServerServices createMockRegionServerService() throws IOException {
2880 return createMockRegionServerService((ServerName)null);
2884 * Create a stubbed out RegionServerService, mainly for getting FS.
2885 * This version is used by TestTokenAuthentication
2887 public RegionServerServices createMockRegionServerService(RpcServerInterface rpc) throws
2888 IOException {
2889 final MockRegionServerServices rss = new MockRegionServerServices(getZooKeeperWatcher());
2890 rss.setFileSystem(getTestFileSystem());
2891 rss.setRpcServer(rpc);
2892 return rss;
2896 * Create a stubbed out RegionServerService, mainly for getting FS.
2897 * This version is used by TestOpenRegionHandler
2899 public RegionServerServices createMockRegionServerService(ServerName name) throws IOException {
2900 final MockRegionServerServices rss = new MockRegionServerServices(getZooKeeperWatcher(), name);
2901 rss.setFileSystem(getTestFileSystem());
2902 return rss;
2906 * Switches the logger for the given class to DEBUG level.
2908 * @param clazz The class for which to switch to debug logging.
2910 public void enableDebug(Class<?> clazz) {
2911 Logger l = LoggerFactory.getLogger(clazz);
2912 if (l instanceof Log4JLogger) {
2913 ((Log4JLogger) l).getLogger().setLevel(org.apache.log4j.Level.DEBUG);
2914 } else if (l instanceof Log4jLoggerAdapter) {
2915 LogManager.getLogger(clazz).setLevel(org.apache.log4j.Level.DEBUG);
2916 } else if (l instanceof Jdk14Logger) {
2917 ((Jdk14Logger) l).getLogger().setLevel(java.util.logging.Level.ALL);
2922 * Expire the Master's session
2923 * @throws Exception
2925 public void expireMasterSession() throws Exception {
2926 HMaster master = getMiniHBaseCluster().getMaster();
2927 expireSession(master.getZooKeeper(), false);
2931 * Expire a region server's session
2932 * @param index which RS
2934 public void expireRegionServerSession(int index) throws Exception {
2935 HRegionServer rs = getMiniHBaseCluster().getRegionServer(index);
2936 expireSession(rs.getZooKeeper(), false);
2937 decrementMinRegionServerCount();
2940 private void decrementMinRegionServerCount() {
2941 // decrement the count for this.conf, for newly spwaned master
2942 // this.hbaseCluster shares this configuration too
2943 decrementMinRegionServerCount(getConfiguration());
2945 // each master thread keeps a copy of configuration
2946 for (MasterThread master : getHBaseCluster().getMasterThreads()) {
2947 decrementMinRegionServerCount(master.getMaster().getConfiguration());
2951 private void decrementMinRegionServerCount(Configuration conf) {
2952 int currentCount = conf.getInt(
2953 ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART, -1);
2954 if (currentCount != -1) {
2955 conf.setInt(ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART,
2956 Math.max(currentCount - 1, 1));
2960 public void expireSession(ZKWatcher nodeZK) throws Exception {
2961 expireSession(nodeZK, false);
2965 * Expire a ZooKeeper session as recommended in ZooKeeper documentation
2966 * http://hbase.apache.org/book.html#trouble.zookeeper
2967 * There are issues when doing this:
2968 * [1] http://www.mail-archive.com/dev@zookeeper.apache.org/msg01942.html
2969 * [2] https://issues.apache.org/jira/browse/ZOOKEEPER-1105
2971 * @param nodeZK - the ZK watcher to expire
2972 * @param checkStatus - true to check if we can create a Table with the
2973 * current configuration.
2975 public void expireSession(ZKWatcher nodeZK, boolean checkStatus)
2976 throws Exception {
2977 Configuration c = new Configuration(this.conf);
2978 String quorumServers = ZKConfig.getZKQuorumServersString(c);
2979 ZooKeeper zk = nodeZK.getRecoverableZooKeeper().getZooKeeper();
2980 byte[] password = zk.getSessionPasswd();
2981 long sessionID = zk.getSessionId();
2983 // Expiry seems to be asynchronous (see comment from P. Hunt in [1]),
2984 // so we create a first watcher to be sure that the
2985 // event was sent. We expect that if our watcher receives the event
2986 // other watchers on the same machine will get is as well.
2987 // When we ask to close the connection, ZK does not close it before
2988 // we receive all the events, so don't have to capture the event, just
2989 // closing the connection should be enough.
2990 ZooKeeper monitor = new ZooKeeper(quorumServers,
2991 1000, new org.apache.zookeeper.Watcher(){
2992 @Override
2993 public void process(WatchedEvent watchedEvent) {
2994 LOG.info("Monitor ZKW received event="+watchedEvent);
2996 } , sessionID, password);
2998 // Making it expire
2999 ZooKeeper newZK = new ZooKeeper(quorumServers,
3000 1000, EmptyWatcher.instance, sessionID, password);
3002 //ensure that we have connection to the server before closing down, otherwise
3003 //the close session event will be eaten out before we start CONNECTING state
3004 long start = System.currentTimeMillis();
3005 while (newZK.getState() != States.CONNECTED
3006 && System.currentTimeMillis() - start < 1000) {
3007 Thread.sleep(1);
3009 newZK.close();
3010 LOG.info("ZK Closed Session 0x" + Long.toHexString(sessionID));
3012 // Now closing & waiting to be sure that the clients get it.
3013 monitor.close();
3015 if (checkStatus) {
3016 getConnection().getTable(TableName.META_TABLE_NAME).close();
3021 * Get the Mini HBase cluster.
3023 * @return hbase cluster
3024 * @see #getHBaseClusterInterface()
3026 public MiniHBaseCluster getHBaseCluster() {
3027 return getMiniHBaseCluster();
3031 * Returns the HBaseCluster instance.
3032 * <p>Returned object can be any of the subclasses of HBaseCluster, and the
3033 * tests referring this should not assume that the cluster is a mini cluster or a
3034 * distributed one. If the test only works on a mini cluster, then specific
3035 * method {@link #getMiniHBaseCluster()} can be used instead w/o the
3036 * need to type-cast.
3038 public HBaseCluster getHBaseClusterInterface() {
3039 //implementation note: we should rename this method as #getHBaseCluster(),
3040 //but this would require refactoring 90+ calls.
3041 return hbaseCluster;
3044 private void initConnection() throws IOException {
3045 User user = UserProvider.instantiate(conf).getCurrent();
3046 this.asyncConnection = ClusterConnectionFactory.createAsyncClusterConnection(conf, null, user);
3050 * Get a Connection to the cluster. Not thread-safe (This class needs a lot of work to make it
3051 * thread-safe).
3052 * @return A Connection that can be shared. Don't close. Will be closed on shutdown of cluster.
3054 public Connection getConnection() throws IOException {
3055 if (this.asyncConnection == null) {
3056 initConnection();
3058 return this.asyncConnection.toConnection();
3061 public AsyncClusterConnection getAsyncConnection() throws IOException {
3062 if (this.asyncConnection == null) {
3063 initConnection();
3065 return this.asyncConnection;
3068 public void closeConnection() throws IOException {
3069 Closeables.close(hbaseAdmin, true);
3070 Closeables.close(asyncConnection, true);
3071 this.hbaseAdmin = null;
3072 this.asyncConnection = null;
3076 * Returns an Admin instance which is shared between HBaseTestingUtility instance users.
3077 * Closing it has no effect, it will be closed automatically when the cluster shutdowns
3079 public synchronized Admin getAdmin() throws IOException {
3080 if (hbaseAdmin == null){
3081 this.hbaseAdmin = getConnection().getAdmin();
3083 return hbaseAdmin;
3086 private Admin hbaseAdmin = null;
3089 * Returns an {@link Hbck} instance. Needs be closed when done.
3091 public Hbck getHbck() throws IOException {
3092 return getConnection().getHbck();
3096 * Unassign the named region.
3098 * @param regionName The region to unassign.
3100 public void unassignRegion(String regionName) throws IOException {
3101 unassignRegion(Bytes.toBytes(regionName));
3105 * Unassign the named region.
3107 * @param regionName The region to unassign.
3109 public void unassignRegion(byte[] regionName) throws IOException {
3110 getAdmin().unassign(regionName, true);
3114 * Closes the region containing the given row.
3116 * @param row The row to find the containing region.
3117 * @param table The table to find the region.
3119 public void unassignRegionByRow(String row, RegionLocator table) throws IOException {
3120 unassignRegionByRow(Bytes.toBytes(row), table);
3124 * Closes the region containing the given row.
3126 * @param row The row to find the containing region.
3127 * @param table The table to find the region.
3128 * @throws IOException
3130 public void unassignRegionByRow(byte[] row, RegionLocator table) throws IOException {
3131 HRegionLocation hrl = table.getRegionLocation(row);
3132 unassignRegion(hrl.getRegion().getRegionName());
3136 * Retrieves a splittable region randomly from tableName
3138 * @param tableName name of table
3139 * @param maxAttempts maximum number of attempts, unlimited for value of -1
3140 * @return the HRegion chosen, null if none was found within limit of maxAttempts
3142 public HRegion getSplittableRegion(TableName tableName, int maxAttempts) {
3143 List<HRegion> regions = getHBaseCluster().getRegions(tableName);
3144 int regCount = regions.size();
3145 Set<Integer> attempted = new HashSet<>();
3146 int idx;
3147 int attempts = 0;
3148 do {
3149 regions = getHBaseCluster().getRegions(tableName);
3150 if (regCount != regions.size()) {
3151 // if there was region movement, clear attempted Set
3152 attempted.clear();
3154 regCount = regions.size();
3155 // There are chances that before we get the region for the table from an RS the region may
3156 // be going for CLOSE. This may be because online schema change is enabled
3157 if (regCount > 0) {
3158 idx = random.nextInt(regCount);
3159 // if we have just tried this region, there is no need to try again
3160 if (attempted.contains(idx))
3161 continue;
3162 try {
3163 regions.get(idx).checkSplit();
3164 return regions.get(idx);
3165 } catch (Exception ex) {
3166 LOG.warn("Caught exception", ex);
3167 attempted.add(idx);
3170 attempts++;
3171 } while (maxAttempts == -1 || attempts < maxAttempts);
3172 return null;
3175 public MiniDFSCluster getDFSCluster() {
3176 return dfsCluster;
3179 public void setDFSCluster(MiniDFSCluster cluster) throws IllegalStateException, IOException {
3180 setDFSCluster(cluster, true);
3184 * Set the MiniDFSCluster
3185 * @param cluster cluster to use
3186 * @param requireDown require the that cluster not be "up" (MiniDFSCluster#isClusterUp) before
3187 * it is set.
3188 * @throws IllegalStateException if the passed cluster is up when it is required to be down
3189 * @throws IOException if the FileSystem could not be set from the passed dfs cluster
3191 public void setDFSCluster(MiniDFSCluster cluster, boolean requireDown)
3192 throws IllegalStateException, IOException {
3193 if (dfsCluster != null && requireDown && dfsCluster.isClusterUp()) {
3194 throw new IllegalStateException("DFSCluster is already running! Shut it down first.");
3196 this.dfsCluster = cluster;
3197 this.setFs();
3200 public FileSystem getTestFileSystem() throws IOException {
3201 return HFileSystem.get(conf);
3205 * Wait until all regions in a table have been assigned. Waits default timeout before giving up
3206 * (30 seconds).
3207 * @param table Table to wait on.
3208 * @throws InterruptedException
3209 * @throws IOException
3211 public void waitTableAvailable(TableName table)
3212 throws InterruptedException, IOException {
3213 waitTableAvailable(table.getName(), 30000);
3216 public void waitTableAvailable(TableName table, long timeoutMillis)
3217 throws InterruptedException, IOException {
3218 waitFor(timeoutMillis, predicateTableAvailable(table));
3222 * Wait until all regions in a table have been assigned
3223 * @param table Table to wait on.
3224 * @param timeoutMillis Timeout.
3226 public void waitTableAvailable(byte[] table, long timeoutMillis)
3227 throws InterruptedException, IOException {
3228 waitFor(timeoutMillis, predicateTableAvailable(TableName.valueOf(table)));
3231 public String explainTableAvailability(TableName tableName) throws IOException {
3232 String msg = explainTableState(tableName, TableState.State.ENABLED) + ", ";
3233 if (getHBaseCluster().getMaster().isAlive()) {
3234 Map<RegionInfo, ServerName> assignments = getHBaseCluster().getMaster().getAssignmentManager()
3235 .getRegionStates().getRegionAssignments();
3236 final List<Pair<RegionInfo, ServerName>> metaLocations =
3237 MetaTableAccessor.getTableRegionsAndLocations(asyncConnection.toConnection(), tableName);
3238 for (Pair<RegionInfo, ServerName> metaLocation : metaLocations) {
3239 RegionInfo hri = metaLocation.getFirst();
3240 ServerName sn = metaLocation.getSecond();
3241 if (!assignments.containsKey(hri)) {
3242 msg += ", region " + hri + " not assigned, but found in meta, it expected to be on " + sn;
3244 } else if (sn == null) {
3245 msg += ", region " + hri + " assigned, but has no server in meta";
3246 } else if (!sn.equals(assignments.get(hri))) {
3247 msg += ", region " + hri + " assigned, but has different servers in meta and AM ( " +
3248 sn + " <> " + assignments.get(hri);
3252 return msg;
3255 public String explainTableState(final TableName table, TableState.State state)
3256 throws IOException {
3257 TableState tableState = MetaTableAccessor.getTableState(asyncConnection.toConnection(), table);
3258 if (tableState == null) {
3259 return "TableState in META: No table state in META for table " + table +
3260 " last state in meta (including deleted is " + findLastTableState(table) + ")";
3261 } else if (!tableState.inStates(state)) {
3262 return "TableState in META: Not " + state + " state, but " + tableState;
3263 } else {
3264 return "TableState in META: OK";
3268 @Nullable
3269 public TableState findLastTableState(final TableName table) throws IOException {
3270 final AtomicReference<TableState> lastTableState = new AtomicReference<>(null);
3271 MetaTableAccessor.Visitor visitor = new MetaTableAccessor.Visitor() {
3272 @Override
3273 public boolean visit(Result r) throws IOException {
3274 if (!Arrays.equals(r.getRow(), table.getName())) {
3275 return false;
3277 TableState state = MetaTableAccessor.getTableState(r);
3278 if (state != null) {
3279 lastTableState.set(state);
3281 return true;
3284 MetaTableAccessor.scanMeta(asyncConnection.toConnection(), null, null,
3285 MetaTableAccessor.QueryType.TABLE, Integer.MAX_VALUE, visitor);
3286 return lastTableState.get();
3290 * Waits for a table to be 'enabled'. Enabled means that table is set as 'enabled' and the
3291 * regions have been all assigned. Will timeout after default period (30 seconds)
3292 * Tolerates nonexistent table.
3293 * @param table the table to wait on.
3294 * @throws InterruptedException if interrupted while waiting
3295 * @throws IOException if an IO problem is encountered
3297 public void waitTableEnabled(TableName table)
3298 throws InterruptedException, IOException {
3299 waitTableEnabled(table, 30000);
3303 * Waits for a table to be 'enabled'. Enabled means that table is set as 'enabled' and the
3304 * regions have been all assigned.
3305 * @see #waitTableEnabled(TableName, long)
3306 * @param table Table to wait on.
3307 * @param timeoutMillis Time to wait on it being marked enabled.
3308 * @throws InterruptedException
3309 * @throws IOException
3311 public void waitTableEnabled(byte[] table, long timeoutMillis)
3312 throws InterruptedException, IOException {
3313 waitTableEnabled(TableName.valueOf(table), timeoutMillis);
3316 public void waitTableEnabled(TableName table, long timeoutMillis)
3317 throws IOException {
3318 waitFor(timeoutMillis, predicateTableEnabled(table));
3322 * Waits for a table to be 'disabled'. Disabled means that table is set as 'disabled'
3323 * Will timeout after default period (30 seconds)
3324 * @param table Table to wait on.
3325 * @throws InterruptedException
3326 * @throws IOException
3328 public void waitTableDisabled(byte[] table)
3329 throws InterruptedException, IOException {
3330 waitTableDisabled(table, 30000);
3333 public void waitTableDisabled(TableName table, long millisTimeout)
3334 throws InterruptedException, IOException {
3335 waitFor(millisTimeout, predicateTableDisabled(table));
3339 * Waits for a table to be 'disabled'. Disabled means that table is set as 'disabled'
3340 * @param table Table to wait on.
3341 * @param timeoutMillis Time to wait on it being marked disabled.
3342 * @throws InterruptedException
3343 * @throws IOException
3345 public void waitTableDisabled(byte[] table, long timeoutMillis)
3346 throws InterruptedException, IOException {
3347 waitTableDisabled(TableName.valueOf(table), timeoutMillis);
3351 * Make sure that at least the specified number of region servers
3352 * are running
3353 * @param num minimum number of region servers that should be running
3354 * @return true if we started some servers
3355 * @throws IOException
3357 public boolean ensureSomeRegionServersAvailable(final int num)
3358 throws IOException {
3359 boolean startedServer = false;
3360 MiniHBaseCluster hbaseCluster = getMiniHBaseCluster();
3361 for (int i=hbaseCluster.getLiveRegionServerThreads().size(); i<num; ++i) {
3362 LOG.info("Started new server=" + hbaseCluster.startRegionServer());
3363 startedServer = true;
3366 return startedServer;
3371 * Make sure that at least the specified number of region servers
3372 * are running. We don't count the ones that are currently stopping or are
3373 * stopped.
3374 * @param num minimum number of region servers that should be running
3375 * @return true if we started some servers
3376 * @throws IOException
3378 public boolean ensureSomeNonStoppedRegionServersAvailable(final int num)
3379 throws IOException {
3380 boolean startedServer = ensureSomeRegionServersAvailable(num);
3382 int nonStoppedServers = 0;
3383 for (JVMClusterUtil.RegionServerThread rst :
3384 getMiniHBaseCluster().getRegionServerThreads()) {
3386 HRegionServer hrs = rst.getRegionServer();
3387 if (hrs.isStopping() || hrs.isStopped()) {
3388 LOG.info("A region server is stopped or stopping:"+hrs);
3389 } else {
3390 nonStoppedServers++;
3393 for (int i=nonStoppedServers; i<num; ++i) {
3394 LOG.info("Started new server=" + getMiniHBaseCluster().startRegionServer());
3395 startedServer = true;
3397 return startedServer;
3402 * This method clones the passed <code>c</code> configuration setting a new
3403 * user into the clone. Use it getting new instances of FileSystem. Only
3404 * works for DistributedFileSystem w/o Kerberos.
3405 * @param c Initial configuration
3406 * @param differentiatingSuffix Suffix to differentiate this user from others.
3407 * @return A new configuration instance with a different user set into it.
3408 * @throws IOException
3410 public static User getDifferentUser(final Configuration c,
3411 final String differentiatingSuffix)
3412 throws IOException {
3413 FileSystem currentfs = FileSystem.get(c);
3414 if (!(currentfs instanceof DistributedFileSystem) || User.isHBaseSecurityEnabled(c)) {
3415 return User.getCurrent();
3417 // Else distributed filesystem. Make a new instance per daemon. Below
3418 // code is taken from the AppendTestUtil over in hdfs.
3419 String username = User.getCurrent().getName() +
3420 differentiatingSuffix;
3421 User user = User.createUserForTesting(c, username,
3422 new String[]{"supergroup"});
3423 return user;
3426 public static NavigableSet<String> getAllOnlineRegions(MiniHBaseCluster cluster)
3427 throws IOException {
3428 NavigableSet<String> online = new TreeSet<>();
3429 for (RegionServerThread rst : cluster.getLiveRegionServerThreads()) {
3430 try {
3431 for (RegionInfo region :
3432 ProtobufUtil.getOnlineRegions(rst.getRegionServer().getRSRpcServices())) {
3433 online.add(region.getRegionNameAsString());
3435 } catch (RegionServerStoppedException e) {
3436 // That's fine.
3439 for (MasterThread mt : cluster.getLiveMasterThreads()) {
3440 try {
3441 for (RegionInfo region :
3442 ProtobufUtil.getOnlineRegions(mt.getMaster().getRSRpcServices())) {
3443 online.add(region.getRegionNameAsString());
3445 } catch (RegionServerStoppedException e) {
3446 // That's fine.
3447 } catch (ServerNotRunningYetException e) {
3448 // That's fine.
3451 return online;
3455 * Set maxRecoveryErrorCount in DFSClient. In 0.20 pre-append its hard-coded to 5 and
3456 * makes tests linger. Here is the exception you'll see:
3457 * <pre>
3458 * 2010-06-15 11:52:28,511 WARN [DataStreamer for file /hbase/.logs/wal.1276627923013 block
3459 * blk_928005470262850423_1021] hdfs.DFSClient$DFSOutputStream(2657): Error Recovery for block
3460 * blk_928005470262850423_1021 failed because recovery from primary datanode 127.0.0.1:53683
3461 * failed 4 times. Pipeline was 127.0.0.1:53687, 127.0.0.1:53683. Will retry...
3462 * </pre>
3463 * @param stream A DFSClient.DFSOutputStream.
3464 * @param max
3465 * @throws NoSuchFieldException
3466 * @throws SecurityException
3467 * @throws IllegalAccessException
3468 * @throws IllegalArgumentException
3470 public static void setMaxRecoveryErrorCount(final OutputStream stream,
3471 final int max) {
3472 try {
3473 Class<?> [] clazzes = DFSClient.class.getDeclaredClasses();
3474 for (Class<?> clazz: clazzes) {
3475 String className = clazz.getSimpleName();
3476 if (className.equals("DFSOutputStream")) {
3477 if (clazz.isInstance(stream)) {
3478 Field maxRecoveryErrorCountField =
3479 stream.getClass().getDeclaredField("maxRecoveryErrorCount");
3480 maxRecoveryErrorCountField.setAccessible(true);
3481 maxRecoveryErrorCountField.setInt(stream, max);
3482 break;
3486 } catch (Exception e) {
3487 LOG.info("Could not set max recovery field", e);
3492 * Uses directly the assignment manager to assign the region. and waits until the specified region
3493 * has completed assignment.
3494 * @return true if the region is assigned false otherwise.
3496 public boolean assignRegion(final RegionInfo regionInfo)
3497 throws IOException, InterruptedException {
3498 final AssignmentManager am = getHBaseCluster().getMaster().getAssignmentManager();
3499 am.assign(regionInfo);
3500 return AssignmentTestingUtil.waitForAssignment(am, regionInfo);
3504 * Move region to destination server and wait till region is completely moved and online
3506 * @param destRegion region to move
3507 * @param destServer destination server of the region
3508 * @throws InterruptedException
3509 * @throws IOException
3511 public void moveRegionAndWait(RegionInfo destRegion, ServerName destServer)
3512 throws InterruptedException, IOException {
3513 HMaster master = getMiniHBaseCluster().getMaster();
3514 // TODO: Here we start the move. The move can take a while.
3515 getAdmin().move(destRegion.getEncodedNameAsBytes(), destServer);
3516 while (true) {
3517 ServerName serverName = master.getAssignmentManager().getRegionStates()
3518 .getRegionServerOfRegion(destRegion);
3519 if (serverName != null && serverName.equals(destServer)) {
3520 assertRegionOnServer(destRegion, serverName, 2000);
3521 break;
3523 Thread.sleep(10);
3528 * Wait until all regions for a table in hbase:meta have a non-empty
3529 * info:server, up to a configuable timeout value (default is 60 seconds)
3530 * This means all regions have been deployed,
3531 * master has been informed and updated hbase:meta with the regions deployed
3532 * server.
3533 * @param tableName the table name
3534 * @throws IOException
3536 public void waitUntilAllRegionsAssigned(final TableName tableName) throws IOException {
3537 waitUntilAllRegionsAssigned(tableName,
3538 this.conf.getLong("hbase.client.sync.wait.timeout.msec", 60000));
3542 * Waith until all system table's regions get assigned
3543 * @throws IOException
3545 public void waitUntilAllSystemRegionsAssigned() throws IOException {
3546 waitUntilAllRegionsAssigned(TableName.META_TABLE_NAME);
3550 * Wait until all regions for a table in hbase:meta have a non-empty
3551 * info:server, or until timeout. This means all regions have been deployed,
3552 * master has been informed and updated hbase:meta with the regions deployed
3553 * server.
3554 * @param tableName the table name
3555 * @param timeout timeout, in milliseconds
3556 * @throws IOException
3558 public void waitUntilAllRegionsAssigned(final TableName tableName, final long timeout)
3559 throws IOException {
3560 if (!TableName.isMetaTableName(tableName)) {
3561 try (final Table meta = getConnection().getTable(TableName.META_TABLE_NAME)) {
3562 LOG.debug("Waiting until all regions of table " + tableName + " get assigned. Timeout = " +
3563 timeout + "ms");
3564 waitFor(timeout, 200, true, new ExplainingPredicate<IOException>() {
3565 @Override
3566 public String explainFailure() throws IOException {
3567 return explainTableAvailability(tableName);
3570 @Override
3571 public boolean evaluate() throws IOException {
3572 Scan scan = new Scan();
3573 scan.addFamily(HConstants.CATALOG_FAMILY);
3574 boolean tableFound = false;
3575 try (ResultScanner s = meta.getScanner(scan)) {
3576 for (Result r; (r = s.next()) != null;) {
3577 byte[] b = r.getValue(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER);
3578 HRegionInfo info = HRegionInfo.parseFromOrNull(b);
3579 if (info != null && info.getTable().equals(tableName)) {
3580 // Get server hosting this region from catalog family. Return false if no server
3581 // hosting this region, or if the server hosting this region was recently killed
3582 // (for fault tolerance testing).
3583 tableFound = true;
3584 byte[] server =
3585 r.getValue(HConstants.CATALOG_FAMILY, HConstants.SERVER_QUALIFIER);
3586 if (server == null) {
3587 return false;
3588 } else {
3589 byte[] startCode =
3590 r.getValue(HConstants.CATALOG_FAMILY, HConstants.STARTCODE_QUALIFIER);
3591 ServerName serverName =
3592 ServerName.valueOf(Bytes.toString(server).replaceFirst(":", ",") + "," +
3593 Bytes.toLong(startCode));
3594 if (!getHBaseClusterInterface().isDistributedCluster() &&
3595 getHBaseCluster().isKilledRS(serverName)) {
3596 return false;
3599 if (RegionStateStore.getRegionState(r,
3600 info.getReplicaId()) != RegionState.State.OPEN) {
3601 return false;
3606 if (!tableFound) {
3607 LOG.warn("Didn't find the entries for table " + tableName + " in meta, already deleted?");
3609 return tableFound;
3614 LOG.info("All regions for table " + tableName + " assigned to meta. Checking AM states.");
3615 // check from the master state if we are using a mini cluster
3616 if (!getHBaseClusterInterface().isDistributedCluster()) {
3617 // So, all regions are in the meta table but make sure master knows of the assignments before
3618 // returning -- sometimes this can lag.
3619 HMaster master = getHBaseCluster().getMaster();
3620 final RegionStates states = master.getAssignmentManager().getRegionStates();
3621 waitFor(timeout, 200, new ExplainingPredicate<IOException>() {
3622 @Override
3623 public String explainFailure() throws IOException {
3624 return explainTableAvailability(tableName);
3627 @Override
3628 public boolean evaluate() throws IOException {
3629 List<RegionInfo> hris = states.getRegionsOfTable(tableName);
3630 return hris != null && !hris.isEmpty();
3634 LOG.info("All regions for table " + tableName + " assigned.");
3638 * Do a small get/scan against one store. This is required because store
3639 * has no actual methods of querying itself, and relies on StoreScanner.
3641 public static List<Cell> getFromStoreFile(HStore store,
3642 Get get) throws IOException {
3643 Scan scan = new Scan(get);
3644 InternalScanner scanner = (InternalScanner) store.getScanner(scan,
3645 scan.getFamilyMap().get(store.getColumnFamilyDescriptor().getName()),
3646 // originally MultiVersionConcurrencyControl.resetThreadReadPoint() was called to set
3647 // readpoint 0.
3650 List<Cell> result = new ArrayList<>();
3651 scanner.next(result);
3652 if (!result.isEmpty()) {
3653 // verify that we are on the row we want:
3654 Cell kv = result.get(0);
3655 if (!CellUtil.matchingRows(kv, get.getRow())) {
3656 result.clear();
3659 scanner.close();
3660 return result;
3664 * Create region split keys between startkey and endKey
3666 * @param startKey
3667 * @param endKey
3668 * @param numRegions the number of regions to be created. it has to be greater than 3.
3669 * @return resulting split keys
3671 public byte[][] getRegionSplitStartKeys(byte[] startKey, byte[] endKey, int numRegions){
3672 assertTrue(numRegions>3);
3673 byte [][] tmpSplitKeys = Bytes.split(startKey, endKey, numRegions - 3);
3674 byte [][] result = new byte[tmpSplitKeys.length+1][];
3675 System.arraycopy(tmpSplitKeys, 0, result, 1, tmpSplitKeys.length);
3676 result[0] = HConstants.EMPTY_BYTE_ARRAY;
3677 return result;
3681 * Do a small get/scan against one store. This is required because store
3682 * has no actual methods of querying itself, and relies on StoreScanner.
3684 public static List<Cell> getFromStoreFile(HStore store,
3685 byte [] row,
3686 NavigableSet<byte[]> columns
3687 ) throws IOException {
3688 Get get = new Get(row);
3689 Map<byte[], NavigableSet<byte[]>> s = get.getFamilyMap();
3690 s.put(store.getColumnFamilyDescriptor().getName(), columns);
3692 return getFromStoreFile(store,get);
3695 public static void assertKVListsEqual(String additionalMsg,
3696 final List<? extends Cell> expected,
3697 final List<? extends Cell> actual) {
3698 final int eLen = expected.size();
3699 final int aLen = actual.size();
3700 final int minLen = Math.min(eLen, aLen);
3702 int i;
3703 for (i = 0; i < minLen
3704 && CellComparator.getInstance().compare(expected.get(i), actual.get(i)) == 0;
3705 ++i) {}
3707 if (additionalMsg == null) {
3708 additionalMsg = "";
3710 if (!additionalMsg.isEmpty()) {
3711 additionalMsg = ". " + additionalMsg;
3714 if (eLen != aLen || i != minLen) {
3715 throw new AssertionError(
3716 "Expected and actual KV arrays differ at position " + i + ": " +
3717 safeGetAsStr(expected, i) + " (length " + eLen +") vs. " +
3718 safeGetAsStr(actual, i) + " (length " + aLen + ")" + additionalMsg);
3722 public static <T> String safeGetAsStr(List<T> lst, int i) {
3723 if (0 <= i && i < lst.size()) {
3724 return lst.get(i).toString();
3725 } else {
3726 return "<out_of_range>";
3730 public String getClusterKey() {
3731 return conf.get(HConstants.ZOOKEEPER_QUORUM) + ":"
3732 + conf.get(HConstants.ZOOKEEPER_CLIENT_PORT) + ":"
3733 + conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
3734 HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
3737 /** Creates a random table with the given parameters */
3738 public Table createRandomTable(TableName tableName,
3739 final Collection<String> families,
3740 final int maxVersions,
3741 final int numColsPerRow,
3742 final int numFlushes,
3743 final int numRegions,
3744 final int numRowsPerFlush)
3745 throws IOException, InterruptedException {
3747 LOG.info("\n\nCreating random table " + tableName + " with " + numRegions +
3748 " regions, " + numFlushes + " storefiles per region, " +
3749 numRowsPerFlush + " rows per flush, maxVersions=" + maxVersions +
3750 "\n");
3752 final Random rand = new Random(tableName.hashCode() * 17L + 12938197137L);
3753 final int numCF = families.size();
3754 final byte[][] cfBytes = new byte[numCF][];
3756 int cfIndex = 0;
3757 for (String cf : families) {
3758 cfBytes[cfIndex++] = Bytes.toBytes(cf);
3762 final int actualStartKey = 0;
3763 final int actualEndKey = Integer.MAX_VALUE;
3764 final int keysPerRegion = (actualEndKey - actualStartKey) / numRegions;
3765 final int splitStartKey = actualStartKey + keysPerRegion;
3766 final int splitEndKey = actualEndKey - keysPerRegion;
3767 final String keyFormat = "%08x";
3768 final Table table = createTable(tableName, cfBytes,
3769 maxVersions,
3770 Bytes.toBytes(String.format(keyFormat, splitStartKey)),
3771 Bytes.toBytes(String.format(keyFormat, splitEndKey)),
3772 numRegions);
3774 if (hbaseCluster != null) {
3775 getMiniHBaseCluster().flushcache(TableName.META_TABLE_NAME);
3778 BufferedMutator mutator = getConnection().getBufferedMutator(tableName);
3780 for (int iFlush = 0; iFlush < numFlushes; ++iFlush) {
3781 for (int iRow = 0; iRow < numRowsPerFlush; ++iRow) {
3782 final byte[] row = Bytes.toBytes(String.format(keyFormat,
3783 actualStartKey + rand.nextInt(actualEndKey - actualStartKey)));
3785 Put put = new Put(row);
3786 Delete del = new Delete(row);
3787 for (int iCol = 0; iCol < numColsPerRow; ++iCol) {
3788 final byte[] cf = cfBytes[rand.nextInt(numCF)];
3789 final long ts = rand.nextInt();
3790 final byte[] qual = Bytes.toBytes("col" + iCol);
3791 if (rand.nextBoolean()) {
3792 final byte[] value = Bytes.toBytes("value_for_row_" + iRow +
3793 "_cf_" + Bytes.toStringBinary(cf) + "_col_" + iCol + "_ts_" +
3794 ts + "_random_" + rand.nextLong());
3795 put.addColumn(cf, qual, ts, value);
3796 } else if (rand.nextDouble() < 0.8) {
3797 del.addColumn(cf, qual, ts);
3798 } else {
3799 del.addColumns(cf, qual, ts);
3803 if (!put.isEmpty()) {
3804 mutator.mutate(put);
3807 if (!del.isEmpty()) {
3808 mutator.mutate(del);
3811 LOG.info("Initiating flush #" + iFlush + " for table " + tableName);
3812 mutator.flush();
3813 if (hbaseCluster != null) {
3814 getMiniHBaseCluster().flushcache(table.getName());
3817 mutator.close();
3819 return table;
3822 private static Random random = new Random();
3824 private static final PortAllocator portAllocator = new PortAllocator(random);
3826 public static int randomFreePort() {
3827 return portAllocator.randomFreePort();
3830 static class PortAllocator {
3831 private static final int MIN_RANDOM_PORT = 0xc000;
3832 private static final int MAX_RANDOM_PORT = 0xfffe;
3834 /** A set of ports that have been claimed using {@link #randomFreePort()}. */
3835 private final Set<Integer> takenRandomPorts = new HashSet<>();
3837 private final Random random;
3838 private final AvailablePortChecker portChecker;
3840 public PortAllocator(Random random) {
3841 this.random = random;
3842 this.portChecker = new AvailablePortChecker() {
3843 @Override
3844 public boolean available(int port) {
3845 try {
3846 ServerSocket sock = new ServerSocket(port);
3847 sock.close();
3848 return true;
3849 } catch (IOException ex) {
3850 return false;
3856 public PortAllocator(Random random, AvailablePortChecker portChecker) {
3857 this.random = random;
3858 this.portChecker = portChecker;
3862 * Returns a random free port and marks that port as taken. Not thread-safe. Expected to be
3863 * called from single-threaded test setup code/
3865 public int randomFreePort() {
3866 int port = 0;
3867 do {
3868 port = randomPort();
3869 if (takenRandomPorts.contains(port)) {
3870 port = 0;
3871 continue;
3873 takenRandomPorts.add(port);
3875 if (!portChecker.available(port)) {
3876 port = 0;
3878 } while (port == 0);
3879 return port;
3883 * Returns a random port. These ports cannot be registered with IANA and are
3884 * intended for dynamic allocation (see http://bit.ly/dynports).
3886 private int randomPort() {
3887 return MIN_RANDOM_PORT
3888 + random.nextInt(MAX_RANDOM_PORT - MIN_RANDOM_PORT);
3891 interface AvailablePortChecker {
3892 boolean available(int port);
3896 public static String randomMultiCastAddress() {
3897 return "226.1.1." + random.nextInt(254);
3900 public static void waitForHostPort(String host, int port)
3901 throws IOException {
3902 final int maxTimeMs = 10000;
3903 final int maxNumAttempts = maxTimeMs / HConstants.SOCKET_RETRY_WAIT_MS;
3904 IOException savedException = null;
3905 LOG.info("Waiting for server at " + host + ":" + port);
3906 for (int attempt = 0; attempt < maxNumAttempts; ++attempt) {
3907 try {
3908 Socket sock = new Socket(InetAddress.getByName(host), port);
3909 sock.close();
3910 savedException = null;
3911 LOG.info("Server at " + host + ":" + port + " is available");
3912 break;
3913 } catch (UnknownHostException e) {
3914 throw new IOException("Failed to look up " + host, e);
3915 } catch (IOException e) {
3916 savedException = e;
3918 Threads.sleepWithoutInterrupt(HConstants.SOCKET_RETRY_WAIT_MS);
3921 if (savedException != null) {
3922 throw savedException;
3927 * Creates a pre-split table for load testing. If the table already exists,
3928 * logs a warning and continues.
3929 * @return the number of regions the table was split into
3931 public static int createPreSplitLoadTestTable(Configuration conf,
3932 TableName tableName, byte[] columnFamily, Algorithm compression,
3933 DataBlockEncoding dataBlockEncoding) throws IOException {
3934 return createPreSplitLoadTestTable(conf, tableName,
3935 columnFamily, compression, dataBlockEncoding, DEFAULT_REGIONS_PER_SERVER, 1,
3936 Durability.USE_DEFAULT);
3939 * Creates a pre-split table for load testing. If the table already exists,
3940 * logs a warning and continues.
3941 * @return the number of regions the table was split into
3943 public static int createPreSplitLoadTestTable(Configuration conf,
3944 TableName tableName, byte[] columnFamily, Algorithm compression,
3945 DataBlockEncoding dataBlockEncoding, int numRegionsPerServer, int regionReplication,
3946 Durability durability)
3947 throws IOException {
3948 HTableDescriptor desc = new HTableDescriptor(tableName);
3949 desc.setDurability(durability);
3950 desc.setRegionReplication(regionReplication);
3951 HColumnDescriptor hcd = new HColumnDescriptor(columnFamily);
3952 hcd.setDataBlockEncoding(dataBlockEncoding);
3953 hcd.setCompressionType(compression);
3954 return createPreSplitLoadTestTable(conf, desc, hcd, numRegionsPerServer);
3958 * Creates a pre-split table for load testing. If the table already exists,
3959 * logs a warning and continues.
3960 * @return the number of regions the table was split into
3962 public static int createPreSplitLoadTestTable(Configuration conf,
3963 TableName tableName, byte[][] columnFamilies, Algorithm compression,
3964 DataBlockEncoding dataBlockEncoding, int numRegionsPerServer, int regionReplication,
3965 Durability durability)
3966 throws IOException {
3967 HTableDescriptor desc = new HTableDescriptor(tableName);
3968 desc.setDurability(durability);
3969 desc.setRegionReplication(regionReplication);
3970 HColumnDescriptor[] hcds = new HColumnDescriptor[columnFamilies.length];
3971 for (int i = 0; i < columnFamilies.length; i++) {
3972 HColumnDescriptor hcd = new HColumnDescriptor(columnFamilies[i]);
3973 hcd.setDataBlockEncoding(dataBlockEncoding);
3974 hcd.setCompressionType(compression);
3975 hcds[i] = hcd;
3977 return createPreSplitLoadTestTable(conf, desc, hcds, numRegionsPerServer);
3981 * Creates a pre-split table for load testing. If the table already exists,
3982 * logs a warning and continues.
3983 * @return the number of regions the table was split into
3985 public static int createPreSplitLoadTestTable(Configuration conf,
3986 TableDescriptor desc, ColumnFamilyDescriptor hcd) throws IOException {
3987 return createPreSplitLoadTestTable(conf, desc, hcd, DEFAULT_REGIONS_PER_SERVER);
3991 * Creates a pre-split table for load testing. If the table already exists,
3992 * logs a warning and continues.
3993 * @return the number of regions the table was split into
3995 public static int createPreSplitLoadTestTable(Configuration conf,
3996 TableDescriptor desc, ColumnFamilyDescriptor hcd, int numRegionsPerServer) throws IOException {
3997 return createPreSplitLoadTestTable(conf, desc, new ColumnFamilyDescriptor[] {hcd},
3998 numRegionsPerServer);
4002 * Creates a pre-split table for load testing. If the table already exists,
4003 * logs a warning and continues.
4004 * @return the number of regions the table was split into
4006 public static int createPreSplitLoadTestTable(Configuration conf,
4007 TableDescriptor desc, ColumnFamilyDescriptor[] hcds,
4008 int numRegionsPerServer) throws IOException {
4009 return createPreSplitLoadTestTable(conf, desc, hcds,
4010 new RegionSplitter.HexStringSplit(), numRegionsPerServer);
4014 * Creates a pre-split table for load testing. If the table already exists,
4015 * logs a warning and continues.
4016 * @return the number of regions the table was split into
4018 public static int createPreSplitLoadTestTable(Configuration conf,
4019 TableDescriptor td, ColumnFamilyDescriptor[] cds,
4020 SplitAlgorithm splitter, int numRegionsPerServer) throws IOException {
4021 TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(td);
4022 for (ColumnFamilyDescriptor cd : cds) {
4023 if (!td.hasColumnFamily(cd.getName())) {
4024 builder.setColumnFamily(cd);
4027 td = builder.build();
4028 int totalNumberOfRegions = 0;
4029 Connection unmanagedConnection = ConnectionFactory.createConnection(conf);
4030 Admin admin = unmanagedConnection.getAdmin();
4032 try {
4033 // create a table a pre-splits regions.
4034 // The number of splits is set as:
4035 // region servers * regions per region server).
4036 int numberOfServers =
4037 admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS)).getLiveServerMetrics()
4038 .size();
4039 if (numberOfServers == 0) {
4040 throw new IllegalStateException("No live regionservers");
4043 totalNumberOfRegions = numberOfServers * numRegionsPerServer;
4044 LOG.info("Number of live regionservers: " + numberOfServers + ", " +
4045 "pre-splitting table into " + totalNumberOfRegions + " regions " +
4046 "(regions per server: " + numRegionsPerServer + ")");
4048 byte[][] splits = splitter.split(
4049 totalNumberOfRegions);
4051 admin.createTable(td, splits);
4052 } catch (MasterNotRunningException e) {
4053 LOG.error("Master not running", e);
4054 throw new IOException(e);
4055 } catch (TableExistsException e) {
4056 LOG.warn("Table " + td.getTableName() +
4057 " already exists, continuing");
4058 } finally {
4059 admin.close();
4060 unmanagedConnection.close();
4062 return totalNumberOfRegions;
4065 public static int getMetaRSPort(Connection connection) throws IOException {
4066 try (RegionLocator locator = connection.getRegionLocator(TableName.META_TABLE_NAME)) {
4067 return locator.getRegionLocation(Bytes.toBytes("")).getPort();
4072 * Due to async racing issue, a region may not be in
4073 * the online region list of a region server yet, after
4074 * the assignment znode is deleted and the new assignment
4075 * is recorded in master.
4077 public void assertRegionOnServer(
4078 final RegionInfo hri, final ServerName server,
4079 final long timeout) throws IOException, InterruptedException {
4080 long timeoutTime = System.currentTimeMillis() + timeout;
4081 while (true) {
4082 List<RegionInfo> regions = getAdmin().getRegions(server);
4083 if (regions.stream().anyMatch(r -> RegionInfo.COMPARATOR.compare(r, hri) == 0)) return;
4084 long now = System.currentTimeMillis();
4085 if (now > timeoutTime) break;
4086 Thread.sleep(10);
4088 fail("Could not find region " + hri.getRegionNameAsString()
4089 + " on server " + server);
4093 * Check to make sure the region is open on the specified
4094 * region server, but not on any other one.
4096 public void assertRegionOnlyOnServer(
4097 final RegionInfo hri, final ServerName server,
4098 final long timeout) throws IOException, InterruptedException {
4099 long timeoutTime = System.currentTimeMillis() + timeout;
4100 while (true) {
4101 List<RegionInfo> regions = getAdmin().getRegions(server);
4102 if (regions.stream().anyMatch(r -> RegionInfo.COMPARATOR.compare(r, hri) == 0)) {
4103 List<JVMClusterUtil.RegionServerThread> rsThreads =
4104 getHBaseCluster().getLiveRegionServerThreads();
4105 for (JVMClusterUtil.RegionServerThread rsThread: rsThreads) {
4106 HRegionServer rs = rsThread.getRegionServer();
4107 if (server.equals(rs.getServerName())) {
4108 continue;
4110 Collection<HRegion> hrs = rs.getOnlineRegionsLocalContext();
4111 for (HRegion r: hrs) {
4112 assertTrue("Region should not be double assigned",
4113 r.getRegionInfo().getRegionId() != hri.getRegionId());
4116 return; // good, we are happy
4118 long now = System.currentTimeMillis();
4119 if (now > timeoutTime) break;
4120 Thread.sleep(10);
4122 fail("Could not find region " + hri.getRegionNameAsString()
4123 + " on server " + server);
4126 public HRegion createTestRegion(String tableName, ColumnFamilyDescriptor cd) throws IOException {
4127 TableDescriptor td =
4128 TableDescriptorBuilder.newBuilder(TableName.valueOf(tableName)).setColumnFamily(cd).build();
4129 RegionInfo info = RegionInfoBuilder.newBuilder(TableName.valueOf(tableName)).build();
4130 return createRegionAndWAL(info, getDataTestDir(), getConfiguration(), td);
4133 public HRegion createTestRegion(String tableName, ColumnFamilyDescriptor cd,
4134 BlockCache blockCache) throws IOException {
4135 TableDescriptor td =
4136 TableDescriptorBuilder.newBuilder(TableName.valueOf(tableName)).setColumnFamily(cd).build();
4137 RegionInfo info = RegionInfoBuilder.newBuilder(TableName.valueOf(tableName)).build();
4138 return createRegionAndWAL(info, getDataTestDir(), getConfiguration(), td, blockCache);
4141 public void setFileSystemURI(String fsURI) {
4142 FS_URI = fsURI;
4146 * Returns a {@link Predicate} for checking that there are no regions in transition in master
4148 public ExplainingPredicate<IOException> predicateNoRegionsInTransition() {
4149 return new ExplainingPredicate<IOException>() {
4150 @Override
4151 public String explainFailure() throws IOException {
4152 final RegionStates regionStates = getMiniHBaseCluster().getMaster()
4153 .getAssignmentManager().getRegionStates();
4154 return "found in transition: " + regionStates.getRegionsInTransition().toString();
4157 @Override
4158 public boolean evaluate() throws IOException {
4159 HMaster master = getMiniHBaseCluster().getMaster();
4160 if (master == null) return false;
4161 AssignmentManager am = master.getAssignmentManager();
4162 if (am == null) return false;
4163 return !am.hasRegionsInTransition();
4169 * Returns a {@link Predicate} for checking that table is enabled
4171 public Waiter.Predicate<IOException> predicateTableEnabled(final TableName tableName) {
4172 return new ExplainingPredicate<IOException>() {
4173 @Override
4174 public String explainFailure() throws IOException {
4175 return explainTableState(tableName, TableState.State.ENABLED);
4178 @Override
4179 public boolean evaluate() throws IOException {
4180 return getAdmin().tableExists(tableName) && getAdmin().isTableEnabled(tableName);
4186 * Returns a {@link Predicate} for checking that table is enabled
4188 public Waiter.Predicate<IOException> predicateTableDisabled(final TableName tableName) {
4189 return new ExplainingPredicate<IOException>() {
4190 @Override
4191 public String explainFailure() throws IOException {
4192 return explainTableState(tableName, TableState.State.DISABLED);
4195 @Override
4196 public boolean evaluate() throws IOException {
4197 return getAdmin().isTableDisabled(tableName);
4203 * Returns a {@link Predicate} for checking that table is enabled
4205 public Waiter.Predicate<IOException> predicateTableAvailable(final TableName tableName) {
4206 return new ExplainingPredicate<IOException>() {
4207 @Override
4208 public String explainFailure() throws IOException {
4209 return explainTableAvailability(tableName);
4212 @Override
4213 public boolean evaluate() throws IOException {
4214 boolean tableAvailable = getAdmin().isTableAvailable(tableName);
4215 if (tableAvailable) {
4216 try (Table table = getConnection().getTable(tableName)) {
4217 TableDescriptor htd = table.getDescriptor();
4218 for (HRegionLocation loc : getConnection().getRegionLocator(tableName)
4219 .getAllRegionLocations()) {
4220 Scan scan = new Scan().withStartRow(loc.getRegion().getStartKey())
4221 .withStopRow(loc.getRegion().getEndKey()).setOneRowLimit()
4222 .setMaxResultsPerColumnFamily(1).setCacheBlocks(false);
4223 for (byte[] family : htd.getColumnFamilyNames()) {
4224 scan.addFamily(family);
4226 try (ResultScanner scanner = table.getScanner(scan)) {
4227 scanner.next();
4232 return tableAvailable;
4238 * Wait until no regions in transition.
4239 * @param timeout How long to wait.
4240 * @throws IOException
4242 public void waitUntilNoRegionsInTransition(final long timeout) throws IOException {
4243 waitFor(timeout, predicateNoRegionsInTransition());
4247 * Wait until no regions in transition. (time limit 15min)
4248 * @throws IOException
4250 public void waitUntilNoRegionsInTransition() throws IOException {
4251 waitUntilNoRegionsInTransition(15 * 60000);
4255 * Wait until labels is ready in VisibilityLabelsCache.
4256 * @param timeoutMillis
4257 * @param labels
4259 public void waitLabelAvailable(long timeoutMillis, final String... labels) {
4260 final VisibilityLabelsCache labelsCache = VisibilityLabelsCache.get();
4261 waitFor(timeoutMillis, new Waiter.ExplainingPredicate<RuntimeException>() {
4263 @Override
4264 public boolean evaluate() {
4265 for (String label : labels) {
4266 if (labelsCache.getLabelOrdinal(label) == 0) {
4267 return false;
4270 return true;
4273 @Override
4274 public String explainFailure() {
4275 for (String label : labels) {
4276 if (labelsCache.getLabelOrdinal(label) == 0) {
4277 return label + " is not available yet";
4280 return "";
4286 * Create a set of column descriptors with the combination of compression,
4287 * encoding, bloom codecs available.
4288 * @return the list of column descriptors
4290 public static List<HColumnDescriptor> generateColumnDescriptors() {
4291 return generateColumnDescriptors("");
4295 * Create a set of column descriptors with the combination of compression,
4296 * encoding, bloom codecs available.
4297 * @param prefix family names prefix
4298 * @return the list of column descriptors
4300 public static List<HColumnDescriptor> generateColumnDescriptors(final String prefix) {
4301 List<HColumnDescriptor> htds = new ArrayList<>();
4302 long familyId = 0;
4303 for (Compression.Algorithm compressionType: getSupportedCompressionAlgorithms()) {
4304 for (DataBlockEncoding encodingType: DataBlockEncoding.values()) {
4305 for (BloomType bloomType: BloomType.values()) {
4306 String name = String.format("%s-cf-!@#&-%d!@#", prefix, familyId);
4307 HColumnDescriptor htd = new HColumnDescriptor(name);
4308 htd.setCompressionType(compressionType);
4309 htd.setDataBlockEncoding(encodingType);
4310 htd.setBloomFilterType(bloomType);
4311 htds.add(htd);
4312 familyId++;
4316 return htds;
4320 * Get supported compression algorithms.
4321 * @return supported compression algorithms.
4323 public static Compression.Algorithm[] getSupportedCompressionAlgorithms() {
4324 String[] allAlgos = HFile.getSupportedCompressionAlgorithms();
4325 List<Compression.Algorithm> supportedAlgos = new ArrayList<>();
4326 for (String algoName : allAlgos) {
4327 try {
4328 Compression.Algorithm algo = Compression.getCompressionAlgorithmByName(algoName);
4329 algo.getCompressor();
4330 supportedAlgos.add(algo);
4331 } catch (Throwable t) {
4332 // this algo is not available
4335 return supportedAlgos.toArray(new Algorithm[supportedAlgos.size()]);
4338 public Result getClosestRowBefore(Region r, byte[] row, byte[] family) throws IOException {
4339 Scan scan = new Scan(row);
4340 scan.setSmall(true);
4341 scan.setCaching(1);
4342 scan.setReversed(true);
4343 scan.addFamily(family);
4344 try (RegionScanner scanner = r.getScanner(scan)) {
4345 List<Cell> cells = new ArrayList<>(1);
4346 scanner.next(cells);
4347 if (r.getRegionInfo().isMetaRegion() && !isTargetTable(row, cells.get(0))) {
4348 return null;
4350 return Result.create(cells);
4354 private boolean isTargetTable(final byte[] inRow, Cell c) {
4355 String inputRowString = Bytes.toString(inRow);
4356 int i = inputRowString.indexOf(HConstants.DELIMITER);
4357 String outputRowString = Bytes.toString(c.getRowArray(), c.getRowOffset(), c.getRowLength());
4358 int o = outputRowString.indexOf(HConstants.DELIMITER);
4359 return inputRowString.substring(0, i).equals(outputRowString.substring(0, o));
4363 * Sets up {@link MiniKdc} for testing security.
4364 * Uses {@link HBaseKerberosUtils} to set the given keytab file as
4365 * {@link HBaseKerberosUtils#KRB_KEYTAB_FILE}.
4367 public MiniKdc setupMiniKdc(File keytabFile) throws Exception {
4368 Properties conf = MiniKdc.createConf();
4369 conf.put(MiniKdc.DEBUG, true);
4370 MiniKdc kdc = null;
4371 File dir = null;
4372 // There is time lag between selecting a port and trying to bind with it. It's possible that
4373 // another service captures the port in between which'll result in BindException.
4374 boolean bindException;
4375 int numTries = 0;
4376 do {
4377 try {
4378 bindException = false;
4379 dir = new File(getDataTestDir("kdc").toUri().getPath());
4380 kdc = new MiniKdc(conf, dir);
4381 kdc.start();
4382 } catch (BindException e) {
4383 FileUtils.deleteDirectory(dir); // clean directory
4384 numTries++;
4385 if (numTries == 3) {
4386 LOG.error("Failed setting up MiniKDC. Tried " + numTries + " times.");
4387 throw e;
4389 LOG.error("BindException encountered when setting up MiniKdc. Trying again.");
4390 bindException = true;
4392 } while (bindException);
4393 HBaseKerberosUtils.setKeytabFileForTesting(keytabFile.getAbsolutePath());
4394 return kdc;
4397 public int getNumHFiles(final TableName tableName, final byte[] family) {
4398 int numHFiles = 0;
4399 for (RegionServerThread regionServerThread : getMiniHBaseCluster().getRegionServerThreads()) {
4400 numHFiles+= getNumHFilesForRS(regionServerThread.getRegionServer(), tableName,
4401 family);
4403 return numHFiles;
4406 public int getNumHFilesForRS(final HRegionServer rs, final TableName tableName,
4407 final byte[] family) {
4408 int numHFiles = 0;
4409 for (Region region : rs.getRegions(tableName)) {
4410 numHFiles += region.getStore(family).getStorefilesCount();
4412 return numHFiles;
4415 public void verifyTableDescriptorIgnoreTableName(TableDescriptor ltd, TableDescriptor rtd) {
4416 assertEquals(ltd.getValues().hashCode(), rtd.getValues().hashCode());
4417 Collection<ColumnFamilyDescriptor> ltdFamilies = Arrays.asList(ltd.getColumnFamilies());
4418 Collection<ColumnFamilyDescriptor> rtdFamilies = Arrays.asList(rtd.getColumnFamilies());
4419 assertEquals(ltdFamilies.size(), rtdFamilies.size());
4420 for (Iterator<ColumnFamilyDescriptor> it = ltdFamilies.iterator(), it2 =
4421 rtdFamilies.iterator(); it.hasNext();) {
4422 assertEquals(0,
4423 ColumnFamilyDescriptor.COMPARATOR.compare(it.next(), it2.next()));