001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.util;
019
020import java.io.IOException;
021import java.math.BigInteger;
022import java.util.Arrays;
023import java.util.Collection;
024import java.util.LinkedList;
025import java.util.List;
026import java.util.Map;
027import java.util.Set;
028import java.util.TreeMap;
029import org.apache.commons.lang3.ArrayUtils;
030import org.apache.commons.lang3.StringUtils;
031import org.apache.hadoop.conf.Configuration;
032import org.apache.hadoop.fs.FSDataInputStream;
033import org.apache.hadoop.fs.FSDataOutputStream;
034import org.apache.hadoop.fs.FileSystem;
035import org.apache.hadoop.fs.Path;
036import org.apache.hadoop.hbase.HBaseConfiguration;
037import org.apache.hadoop.hbase.HConstants;
038import org.apache.hadoop.hbase.HRegionInfo;
039import org.apache.hadoop.hbase.HRegionLocation;
040import org.apache.hadoop.hbase.ServerName;
041import org.apache.hadoop.hbase.TableName;
042import org.apache.hadoop.hbase.client.Admin;
043import org.apache.hadoop.hbase.client.ClusterConnection;
044import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
045import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
046import org.apache.hadoop.hbase.client.Connection;
047import org.apache.hadoop.hbase.client.ConnectionFactory;
048import org.apache.hadoop.hbase.client.NoServerForRegionException;
049import org.apache.hadoop.hbase.client.RegionLocator;
050import org.apache.hadoop.hbase.client.Table;
051import org.apache.hadoop.hbase.client.TableDescriptor;
052import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
053import org.apache.hadoop.hbase.regionserver.HRegionFileSystem;
054import org.apache.yetus.audience.InterfaceAudience;
055import org.slf4j.Logger;
056import org.slf4j.LoggerFactory;
057
058import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
059import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
060import org.apache.hbase.thirdparty.com.google.common.collect.Maps;
061import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
062import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLine;
063import org.apache.hbase.thirdparty.org.apache.commons.cli.GnuParser;
064import org.apache.hbase.thirdparty.org.apache.commons.cli.HelpFormatter;
065import org.apache.hbase.thirdparty.org.apache.commons.cli.OptionBuilder;
066import org.apache.hbase.thirdparty.org.apache.commons.cli.Options;
067import org.apache.hbase.thirdparty.org.apache.commons.cli.ParseException;
068
069/**
070 * The {@link RegionSplitter} class provides several utilities to help in the administration
071 * lifecycle for developers who choose to manually split regions instead of having HBase handle that
072 * automatically. The most useful utilities are:
073 * <p>
074 * <ul>
075 * <li>Create a table with a specified number of pre-split regions
076 * <li>Execute a rolling split of all regions on an existing table
077 * </ul>
078 * <p>
079 * Both operations can be safely done on a live server.
080 * <p>
081 * <b>Question:</b> How do I turn off automatic splitting? <br>
082 * <b>Answer:</b> Automatic splitting is determined by the configuration value
083 * <i>HConstants.HREGION_MAX_FILESIZE</i>. It is not recommended that you set this to Long.MAX_VALUE
084 * in case you forget about manual splits. A suggested setting is 100GB, which would result in &gt;
085 * 1hr major compactions if reached.
086 * <p>
087 * <b>Question:</b> Why did the original authors decide to manually split? <br>
088 * <b>Answer:</b> Specific workload characteristics of our use case allowed us to benefit from a
089 * manual split system.
090 * <p>
091 * <ul>
092 * <li>Data (~1k) that would grow instead of being replaced
093 * <li>Data growth was roughly uniform across all regions
094 * <li>OLTP workload. Data loss is a big deal.
095 * </ul>
096 * <p>
097 * <b>Question:</b> Why is manual splitting good for this workload? <br>
098 * <b>Answer:</b> Although automated splitting is not a bad option, there are benefits to manual
099 * splitting.
100 * <p>
101 * <ul>
102 * <li>With growing amounts of data, splits will continually be needed. Since you always know
103 * exactly what regions you have, long-term debugging and profiling is much easier with manual
104 * splits. It is hard to trace the logs to understand region level problems if it keeps splitting
105 * and getting renamed.
106 * <li>Data offlining bugs + unknown number of split regions == oh crap! If an WAL or StoreFile was
107 * mistakenly unprocessed by HBase due to a weird bug and you notice it a day or so later, you can
108 * be assured that the regions specified in these files are the same as the current regions and you
109 * have less headaches trying to restore/replay your data.
110 * <li>You can finely tune your compaction algorithm. With roughly uniform data growth, it's easy to
111 * cause split / compaction storms as the regions all roughly hit the same data size at the same
112 * time. With manual splits, you can let staggered, time-based major compactions spread out your
113 * network IO load.
114 * </ul>
115 * <p>
116 * <b>Question:</b> What's the optimal number of pre-split regions to create? <br>
117 * <b>Answer:</b> Mileage will vary depending upon your application.
118 * <p>
119 * The short answer for our application is that we started with 10 pre-split regions / server and
120 * watched our data growth over time. It's better to err on the side of too little regions and
121 * rolling split later.
122 * <p>
123 * The more complicated answer is that this depends upon the largest storefile in your region. With
124 * a growing data size, this will get larger over time. You want the largest region to be just big
125 * enough that the {@link org.apache.hadoop.hbase.regionserver.HStore} compact selection algorithm
126 * only compacts it due to a timed major. If you don't, your cluster can be prone to compaction
127 * storms as the algorithm decides to run major compactions on a large series of regions all at
128 * once. Note that compaction storms are due to the uniform data growth, not the manual split
129 * decision.
130 * <p>
131 * If you pre-split your regions too thin, you can increase the major compaction interval by
132 * configuring HConstants.MAJOR_COMPACTION_PERIOD. If your data size grows too large, use this
133 * script to perform a network IO safe rolling split of all regions.
134 */
135@InterfaceAudience.Private
136public class RegionSplitter {
137  private static final Logger LOG = LoggerFactory.getLogger(RegionSplitter.class);
138
139  /**
140   * A generic interface for the RegionSplitter code to use for all it's functionality. Note that
141   * the original authors of this code use {@link HexStringSplit} to partition their table and set
142   * it as default, but provided this for your custom algorithm. To use, create a new derived class
143   * from this interface and call {@link RegionSplitter#createPresplitTable} or
144   * RegionSplitter#rollingSplit(TableName, SplitAlgorithm, Configuration) with the argument
145   * splitClassName giving the name of your class.
146   */
147  public interface SplitAlgorithm {
148    /**
149     * Split a pre-existing region into 2 regions. first row (inclusive) last row (exclusive)
150     * @return the split row to use
151     */
152    byte[] split(byte[] start, byte[] end);
153
154    /**
155     * Split an entire table. number of regions to split the table into user input is validated at
156     * this time. may throw a runtime exception in response to a parse failure
157     * @return array of split keys for the initial regions of the table. The length of the returned
158     *         array should be numRegions-1.
159     */
160    byte[][] split(int numRegions);
161
162    /**
163     * Some MapReduce jobs may want to run multiple mappers per region, this is intended for such
164     * usecase.
165     * @param start     first row (inclusive)
166     * @param end       last row (exclusive)
167     * @param numSplits number of splits to generate
168     * @param inclusive whether start and end are returned as split points
169     */
170    byte[][] split(byte[] start, byte[] end, int numSplits, boolean inclusive);
171
172    /**
173     * In HBase, the first row is represented by an empty byte array. This might cause problems with
174     * your split algorithm or row printing. All your APIs will be passed firstRow() instead of
175     * empty array.
176     * @return your representation of your first row
177     */
178    byte[] firstRow();
179
180    /**
181     * In HBase, the last row is represented by an empty byte array. This might cause problems with
182     * your split algorithm or row printing. All your APIs will be passed firstRow() instead of
183     * empty array.
184     * @return your representation of your last row
185     */
186    byte[] lastRow();
187
188    /**
189     * In HBase, the last row is represented by an empty byte array. Set this value to help the
190     * split code understand how to evenly divide the first region. raw user input (may throw
191     * RuntimeException on parse failure)
192     */
193    void setFirstRow(String userInput);
194
195    /**
196     * In HBase, the last row is represented by an empty byte array. Set this value to help the
197     * split code understand how to evenly divide the last region. Note that this last row is
198     * inclusive for all rows sharing the same prefix. raw user input (may throw RuntimeException on
199     * parse failure)
200     */
201    void setLastRow(String userInput);
202
203    /**
204     * user or file input for row
205     * @return byte array representation of this row for HBase
206     */
207    byte[] strToRow(String input);
208
209    /**
210     * byte array representing a row in HBase
211     * @return String to use for debug &amp; file printing
212     */
213    String rowToStr(byte[] row);
214
215    /** Returns the separator character to use when storing / printing the row */
216    String separator();
217
218    /**
219     * Set the first row
220     * @param userInput byte array of the row key.
221     */
222    void setFirstRow(byte[] userInput);
223
224    /**
225     * Set the last row
226     * @param userInput byte array of the row key.
227     */
228    void setLastRow(byte[] userInput);
229  }
230
231  /**
232   * The main function for the RegionSplitter application. Common uses:
233   * <p>
234   * <ul>
235   * <li>create a table named 'myTable' with 60 pre-split regions containing 2 column families
236   * 'test' &amp; 'rs', assuming the keys are hex-encoded ASCII:
237   * <ul>
238   * <li>bin/hbase org.apache.hadoop.hbase.util.RegionSplitter -c 60 -f test:rs myTable
239   * HexStringSplit
240   * </ul>
241   * <li>create a table named 'myTable' with 50 pre-split regions, assuming the keys are
242   * decimal-encoded ASCII:
243   * <ul>
244   * <li>bin/hbase org.apache.hadoop.hbase.util.RegionSplitter -c 50 myTable DecimalStringSplit
245   * </ul>
246   * <li>perform a rolling split of 'myTable' (i.e. 60 =&gt; 120 regions), # 2 outstanding splits at
247   * a time, assuming keys are uniformly distributed bytes:
248   * <ul>
249   * <li>bin/hbase org.apache.hadoop.hbase.util.RegionSplitter -r -o 2 myTable UniformSplit
250   * </ul>
251   * </ul>
252   * There are three SplitAlgorithms built into RegionSplitter, HexStringSplit, DecimalStringSplit,
253   * and UniformSplit. These are different strategies for choosing region boundaries. See their
254   * source code for details. Usage: RegionSplitter &lt;TABLE&gt; &lt;SPLITALGORITHM&gt; &lt;-c
255   * &lt;# regions&gt; -f &lt;family:family:...&gt; | -r [-o &lt;# outstanding splits&gt;]&gt; [-D
256   * &lt;conf.param=value&gt;] HBase IO problem user requested exit problem parsing user input
257   */
258  @SuppressWarnings("static-access")
259  public static void main(String[] args) throws IOException, InterruptedException, ParseException {
260    Configuration conf = HBaseConfiguration.create();
261
262    // parse user input
263    Options opt = new Options();
264    opt.addOption(OptionBuilder.withArgName("property=value").hasArg()
265      .withDescription("Override HBase Configuration Settings").create("D"));
266    opt.addOption(OptionBuilder.withArgName("region count").hasArg()
267      .withDescription("Create a new table with a pre-split number of regions").create("c"));
268    opt.addOption(OptionBuilder.withArgName("family:family:...").hasArg()
269      .withDescription("Column Families to create with new table.  Required with -c").create("f"));
270    opt.addOption("h", false, "Print this usage help");
271    opt.addOption("r", false, "Perform a rolling split of an existing region");
272    opt.addOption(OptionBuilder.withArgName("count").hasArg()
273      .withDescription("Max outstanding splits that have unfinished major compactions")
274      .create("o"));
275    opt.addOption(null, "firstrow", true, "First Row in Table for Split Algorithm");
276    opt.addOption(null, "lastrow", true, "Last Row in Table for Split Algorithm");
277    opt.addOption(null, "risky", false, "Skip verification steps to complete quickly. "
278      + "STRONGLY DISCOURAGED for production systems.  ");
279    CommandLine cmd = new GnuParser().parse(opt, args);
280
281    if (cmd.hasOption("D")) {
282      for (String confOpt : cmd.getOptionValues("D")) {
283        String[] kv = confOpt.split("=", 2);
284        if (kv.length == 2) {
285          conf.set(kv[0], kv[1]);
286          LOG.debug("-D configuration override: " + kv[0] + "=" + kv[1]);
287        } else {
288          throw new ParseException("-D option format invalid: " + confOpt);
289        }
290      }
291    }
292
293    if (cmd.hasOption("risky")) {
294      conf.setBoolean("split.verify", false);
295    }
296
297    boolean createTable = cmd.hasOption("c") && cmd.hasOption("f");
298    boolean rollingSplit = cmd.hasOption("r");
299    boolean oneOperOnly = createTable ^ rollingSplit;
300
301    if (2 != cmd.getArgList().size() || !oneOperOnly || cmd.hasOption("h")) {
302      new HelpFormatter().printHelp("bin/hbase regionsplitter <TABLE> <SPLITALGORITHM>\n"
303        + "SPLITALGORITHM is the java class name of a class implementing "
304        + "SplitAlgorithm, or one of the special strings HexStringSplit or "
305        + "DecimalStringSplit or UniformSplit, which are built-in split algorithms. "
306        + "HexStringSplit treats keys as hexadecimal ASCII, and "
307        + "DecimalStringSplit treats keys as decimal ASCII, and "
308        + "UniformSplit treats keys as arbitrary bytes.", opt);
309      return;
310    }
311    TableName tableName = TableName.valueOf(cmd.getArgs()[0]);
312    String splitClass = cmd.getArgs()[1];
313    SplitAlgorithm splitAlgo = newSplitAlgoInstance(conf, splitClass);
314
315    if (cmd.hasOption("firstrow")) {
316      splitAlgo.setFirstRow(cmd.getOptionValue("firstrow"));
317    }
318    if (cmd.hasOption("lastrow")) {
319      splitAlgo.setLastRow(cmd.getOptionValue("lastrow"));
320    }
321
322    if (createTable) {
323      conf.set("split.count", cmd.getOptionValue("c"));
324      createPresplitTable(tableName, splitAlgo, cmd.getOptionValue("f").split(":"), conf);
325    }
326
327    if (rollingSplit) {
328      if (cmd.hasOption("o")) {
329        conf.set("split.outstanding", cmd.getOptionValue("o"));
330      }
331      rollingSplit(tableName, splitAlgo, conf);
332    }
333  }
334
335  static void createPresplitTable(TableName tableName, SplitAlgorithm splitAlgo,
336    String[] columnFamilies, Configuration conf) throws IOException, InterruptedException {
337    final int splitCount = conf.getInt("split.count", 0);
338    Preconditions.checkArgument(splitCount > 1, "Split count must be > 1");
339
340    Preconditions.checkArgument(columnFamilies.length > 0,
341      "Must specify at least one column family. ");
342    LOG.debug("Creating table " + tableName + " with " + columnFamilies.length
343      + " column families.  Presplitting to " + splitCount + " regions");
344
345    TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(tableName);
346    for (String cf : columnFamilies) {
347      builder.setColumnFamily(ColumnFamilyDescriptorBuilder.of(cf));
348    }
349    try (Connection connection = ConnectionFactory.createConnection(conf)) {
350      Admin admin = connection.getAdmin();
351      try {
352        Preconditions.checkArgument(!admin.tableExists(tableName),
353          "Table already exists: " + tableName);
354        admin.createTable(builder.build(), splitAlgo.split(splitCount));
355      } finally {
356        admin.close();
357      }
358      LOG.debug("Table created!  Waiting for regions to show online in META...");
359      if (!conf.getBoolean("split.verify", true)) {
360        // NOTE: createTable is synchronous on the table, but not on the regions
361        int onlineRegions = 0;
362        try (RegionLocator locator = connection.getRegionLocator(tableName)) {
363          while (onlineRegions < splitCount) {
364            onlineRegions = locator.getAllRegionLocations().size();
365            LOG.debug(onlineRegions + " of " + splitCount + " regions online...");
366            if (onlineRegions < splitCount) {
367              Thread.sleep(10 * 1000); // sleep
368            }
369          }
370        }
371      }
372      LOG.debug("Finished creating table with " + splitCount + " regions");
373    }
374  }
375
376  /**
377   * Alternative getCurrentNrHRS which is no longer available.
378   * @return Rough count of regionservers out on cluster.
379   * @throws IOException if a remote or network exception occurs
380   */
381  private static int getRegionServerCount(final Connection connection) throws IOException {
382    try (Admin admin = connection.getAdmin()) {
383      Collection<ServerName> servers = admin.getRegionServers();
384      return servers == null || servers.isEmpty() ? 0 : servers.size();
385    }
386  }
387
388  private static byte[] readFile(final FileSystem fs, final Path path) throws IOException {
389    FSDataInputStream tmpIn = fs.open(path);
390    try {
391      byte[] rawData = new byte[tmpIn.available()];
392      tmpIn.readFully(rawData);
393      return rawData;
394    } finally {
395      tmpIn.close();
396    }
397  }
398
399  static void rollingSplit(TableName tableName, SplitAlgorithm splitAlgo, Configuration conf)
400    throws IOException, InterruptedException {
401    final int minOS = conf.getInt("split.outstanding", 2);
402    try (Connection connection = ConnectionFactory.createConnection(conf)) {
403      // Max outstanding splits. default == 50% of servers
404      final int MAX_OUTSTANDING = Math.max(getRegionServerCount(connection) / 2, minOS);
405
406      Path hbDir = CommonFSUtils.getRootDir(conf);
407      Path tableDir = CommonFSUtils.getTableDir(hbDir, tableName);
408      Path splitFile = new Path(tableDir, "_balancedSplit");
409      FileSystem fs = FileSystem.get(conf);
410
411      // Get a list of daughter regions to create
412      LinkedList<Pair<byte[], byte[]>> tmpRegionSet = null;
413      try (Table table = connection.getTable(tableName)) {
414        tmpRegionSet = getSplits(connection, tableName, splitAlgo);
415      }
416      LinkedList<Pair<byte[], byte[]>> outstanding = Lists.newLinkedList();
417      int splitCount = 0;
418      final int origCount = tmpRegionSet.size();
419
420      // all splits must compact & we have 1 compact thread, so 2 split
421      // requests to the same RS can stall the outstanding split queue.
422      // To fix, group the regions into an RS pool and round-robin through it
423      LOG.debug("Bucketing regions by regionserver...");
424      TreeMap<ServerName, LinkedList<Pair<byte[], byte[]>>> daughterRegions = Maps.newTreeMap();
425      // Get a regionLocator. Need it in below.
426      try (RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
427        for (Pair<byte[], byte[]> dr : tmpRegionSet) {
428          ServerName rsLocation = regionLocator.getRegionLocation(dr.getSecond()).getServerName();
429          if (!daughterRegions.containsKey(rsLocation)) {
430            LinkedList<Pair<byte[], byte[]>> entry = Lists.newLinkedList();
431            daughterRegions.put(rsLocation, entry);
432          }
433          daughterRegions.get(rsLocation).add(dr);
434        }
435        LOG.debug("Done with bucketing.  Split time!");
436        long startTime = EnvironmentEdgeManager.currentTime();
437
438        // Open the split file and modify it as splits finish
439        byte[] rawData = readFile(fs, splitFile);
440
441        FSDataOutputStream splitOut = fs.create(splitFile);
442        try {
443          splitOut.write(rawData);
444
445          try {
446            // *** split code ***
447            while (!daughterRegions.isEmpty()) {
448              LOG.debug(daughterRegions.size() + " RS have regions to splt.");
449
450              // Get ServerName to region count mapping
451              final TreeMap<ServerName, Integer> rsSizes = Maps.newTreeMap();
452              List<HRegionLocation> hrls = regionLocator.getAllRegionLocations();
453              for (HRegionLocation hrl : hrls) {
454                ServerName sn = hrl.getServerName();
455                if (rsSizes.containsKey(sn)) {
456                  rsSizes.put(sn, rsSizes.get(sn) + 1);
457                } else {
458                  rsSizes.put(sn, 1);
459                }
460              }
461
462              // Round-robin through the ServerName list. Choose the lightest-loaded servers
463              // first to keep the master from load-balancing regions as we split.
464              for (Map.Entry<ServerName,
465                LinkedList<Pair<byte[], byte[]>>> daughterRegion : daughterRegions.entrySet()) {
466                Pair<byte[], byte[]> dr = null;
467                ServerName rsLoc = daughterRegion.getKey();
468                LinkedList<Pair<byte[], byte[]>> regionList = daughterRegion.getValue();
469
470                // Find a region in the ServerName list that hasn't been moved
471                LOG.debug("Finding a region on " + rsLoc);
472                while (!regionList.isEmpty()) {
473                  dr = regionList.pop();
474
475                  // get current region info
476                  byte[] split = dr.getSecond();
477                  HRegionLocation regionLoc = regionLocator.getRegionLocation(split);
478
479                  // if this region moved locations
480                  ServerName newRs = regionLoc.getServerName();
481                  if (newRs.compareTo(rsLoc) != 0) {
482                    LOG.debug("Region with " + splitAlgo.rowToStr(split) + " moved to " + newRs
483                      + ". Relocating...");
484                    // relocate it, don't use it right now
485                    if (!daughterRegions.containsKey(newRs)) {
486                      LinkedList<Pair<byte[], byte[]>> entry = Lists.newLinkedList();
487                      daughterRegions.put(newRs, entry);
488                    }
489                    daughterRegions.get(newRs).add(dr);
490                    dr = null;
491                    continue;
492                  }
493
494                  // make sure this region wasn't already split
495                  byte[] sk = regionLoc.getRegionInfo().getStartKey();
496                  if (sk.length != 0) {
497                    if (Bytes.equals(split, sk)) {
498                      LOG.debug("Region already split on " + splitAlgo.rowToStr(split)
499                        + ".  Skipping this region...");
500                      ++splitCount;
501                      dr = null;
502                      continue;
503                    }
504                    byte[] start = dr.getFirst();
505                    Preconditions.checkArgument(Bytes.equals(start, sk),
506                      splitAlgo.rowToStr(start) + " != " + splitAlgo.rowToStr(sk));
507                  }
508
509                  // passed all checks! found a good region
510                  break;
511                }
512                if (regionList.isEmpty()) {
513                  daughterRegions.remove(rsLoc);
514                }
515                if (dr == null) continue;
516
517                // we have a good region, time to split!
518                byte[] split = dr.getSecond();
519                LOG.debug("Splitting at " + splitAlgo.rowToStr(split));
520                try (Admin admin = connection.getAdmin()) {
521                  admin.split(tableName, split);
522                }
523
524                LinkedList<Pair<byte[], byte[]>> finished = Lists.newLinkedList();
525                LinkedList<Pair<byte[], byte[]>> local_finished = Lists.newLinkedList();
526                if (conf.getBoolean("split.verify", true)) {
527                  // we need to verify and rate-limit our splits
528                  outstanding.addLast(dr);
529                  // with too many outstanding splits, wait for some to finish
530                  while (outstanding.size() >= MAX_OUTSTANDING) {
531                    LOG.debug("Wait for outstanding splits " + outstanding.size());
532                    local_finished = splitScan(outstanding, connection, tableName, splitAlgo);
533                    if (local_finished.isEmpty()) {
534                      Thread.sleep(30 * 1000);
535                    } else {
536                      finished.addAll(local_finished);
537                      outstanding.removeAll(local_finished);
538                      LOG.debug(local_finished.size() + " outstanding splits finished");
539                    }
540                  }
541                } else {
542                  finished.add(dr);
543                }
544
545                // mark each finished region as successfully split.
546                for (Pair<byte[], byte[]> region : finished) {
547                  splitOut.writeChars("- " + splitAlgo.rowToStr(region.getFirst()) + " "
548                    + splitAlgo.rowToStr(region.getSecond()) + "\n");
549                  splitCount++;
550                  if (splitCount % 10 == 0) {
551                    long tDiff = (EnvironmentEdgeManager.currentTime() - startTime) / splitCount;
552                    LOG.debug(
553                      "STATUS UPDATE: " + splitCount + " / " + origCount + ". Avg Time / Split = "
554                        + org.apache.hadoop.util.StringUtils.formatTime(tDiff));
555                  }
556                }
557              }
558            }
559            if (conf.getBoolean("split.verify", true)) {
560              while (!outstanding.isEmpty()) {
561                LOG.debug("Finally Wait for outstanding splits " + outstanding.size());
562                LinkedList<Pair<byte[], byte[]>> finished =
563                  splitScan(outstanding, connection, tableName, splitAlgo);
564                if (finished.isEmpty()) {
565                  Thread.sleep(30 * 1000);
566                } else {
567                  outstanding.removeAll(finished);
568                  for (Pair<byte[], byte[]> region : finished) {
569                    splitOut.writeChars("- " + splitAlgo.rowToStr(region.getFirst()) + " "
570                      + splitAlgo.rowToStr(region.getSecond()) + "\n");
571                    splitCount++;
572                  }
573                  LOG.debug("Finally " + finished.size() + " outstanding splits finished");
574                }
575              }
576            }
577            LOG.debug("All regions have been successfully split!");
578          } finally {
579            long tDiff = EnvironmentEdgeManager.currentTime() - startTime;
580            LOG.debug("TOTAL TIME = " + org.apache.hadoop.util.StringUtils.formatTime(tDiff));
581            LOG.debug("Splits = " + splitCount);
582            if (0 < splitCount) {
583              LOG.debug("Avg Time / Split = "
584                + org.apache.hadoop.util.StringUtils.formatTime(tDiff / splitCount));
585            }
586          }
587        } finally {
588          splitOut.close();
589          fs.delete(splitFile, false);
590        }
591      }
592    }
593  }
594
595  /**
596   * @throws IOException if the specified SplitAlgorithm class couldn't be instantiated
597   */
598  public static SplitAlgorithm newSplitAlgoInstance(Configuration conf, String splitClassName)
599    throws IOException {
600    Class<?> splitClass;
601
602    // For split algorithms builtin to RegionSplitter, the user can specify
603    // their simple class name instead of a fully qualified class name.
604    if (splitClassName.equals(HexStringSplit.class.getSimpleName())) {
605      splitClass = HexStringSplit.class;
606    } else if (splitClassName.equals(DecimalStringSplit.class.getSimpleName())) {
607      splitClass = DecimalStringSplit.class;
608    } else if (splitClassName.equals(UniformSplit.class.getSimpleName())) {
609      splitClass = UniformSplit.class;
610    } else {
611      try {
612        splitClass = conf.getClassByName(splitClassName);
613      } catch (ClassNotFoundException e) {
614        throw new IOException("Couldn't load split class " + splitClassName, e);
615      }
616      if (splitClass == null) {
617        throw new IOException("Failed loading split class " + splitClassName);
618      }
619      if (!SplitAlgorithm.class.isAssignableFrom(splitClass)) {
620        throw new IOException("Specified split class doesn't implement SplitAlgorithm");
621      }
622    }
623    try {
624      return splitClass.asSubclass(SplitAlgorithm.class).getDeclaredConstructor().newInstance();
625    } catch (Exception e) {
626      throw new IOException("Problem loading split algorithm: ", e);
627    }
628  }
629
630  static LinkedList<Pair<byte[], byte[]>> splitScan(LinkedList<Pair<byte[], byte[]>> regionList,
631    final Connection connection, final TableName tableName, SplitAlgorithm splitAlgo)
632    throws IOException, InterruptedException {
633    LinkedList<Pair<byte[], byte[]>> finished = Lists.newLinkedList();
634    LinkedList<Pair<byte[], byte[]>> logicalSplitting = Lists.newLinkedList();
635    LinkedList<Pair<byte[], byte[]>> physicalSplitting = Lists.newLinkedList();
636
637    // Get table info
638    Pair<Path, Path> tableDirAndSplitFile =
639      getTableDirAndSplitFile(connection.getConfiguration(), tableName);
640    Path tableDir = tableDirAndSplitFile.getFirst();
641    FileSystem fs = tableDir.getFileSystem(connection.getConfiguration());
642    // Clear the cache to forcibly refresh region information
643    ((ClusterConnection) connection).clearRegionLocationCache();
644    TableDescriptor htd = null;
645    try (Table table = connection.getTable(tableName)) {
646      htd = table.getDescriptor();
647    }
648    try (RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
649
650      // for every region that hasn't been verified as a finished split
651      for (Pair<byte[], byte[]> region : regionList) {
652        byte[] start = region.getFirst();
653        byte[] split = region.getSecond();
654
655        // see if the new split daughter region has come online
656        try {
657          HRegionInfo dri = regionLocator.getRegionLocation(split).getRegionInfo();
658          if (dri.isOffline() || !Bytes.equals(dri.getStartKey(), split)) {
659            logicalSplitting.add(region);
660            continue;
661          }
662        } catch (NoServerForRegionException nsfre) {
663          // NSFRE will occur if the old hbase:meta entry has no server assigned
664          LOG.info(nsfre.toString(), nsfre);
665          logicalSplitting.add(region);
666          continue;
667        }
668
669        try {
670          // when a daughter region is opened, a compaction is triggered
671          // wait until compaction completes for both daughter regions
672          LinkedList<HRegionInfo> check = Lists.newLinkedList();
673          check.add(regionLocator.getRegionLocation(start).getRegionInfo());
674          check.add(regionLocator.getRegionLocation(split).getRegionInfo());
675          for (HRegionInfo hri : check.toArray(new HRegionInfo[check.size()])) {
676            byte[] sk = hri.getStartKey();
677            if (sk.length == 0) sk = splitAlgo.firstRow();
678
679            HRegionFileSystem regionFs = HRegionFileSystem
680              .openRegionFromFileSystem(connection.getConfiguration(), fs, tableDir, hri, true);
681
682            // Check every Column Family for that region -- check does not have references.
683            boolean refFound = false;
684            for (ColumnFamilyDescriptor c : htd.getColumnFamilies()) {
685              if ((refFound = regionFs.hasReferences(c.getNameAsString()))) {
686                break;
687              }
688            }
689
690            // compaction is completed when all reference files are gone
691            if (!refFound) {
692              check.remove(hri);
693            }
694          }
695          if (check.isEmpty()) {
696            finished.add(region);
697          } else {
698            physicalSplitting.add(region);
699          }
700        } catch (NoServerForRegionException nsfre) {
701          LOG.debug("No Server Exception thrown for: " + splitAlgo.rowToStr(start));
702          physicalSplitting.add(region);
703          ((ClusterConnection) connection).clearRegionLocationCache();
704        }
705      }
706
707      LOG.debug("Split Scan: " + finished.size() + " finished / " + logicalSplitting.size()
708        + " split wait / " + physicalSplitting.size() + " reference wait");
709
710      return finished;
711    }
712  }
713
714  /**
715   * @return A Pair where first item is table dir and second is the split file.
716   * @throws IOException if a remote or network exception occurs
717   */
718  private static Pair<Path, Path> getTableDirAndSplitFile(final Configuration conf,
719    final TableName tableName) throws IOException {
720    Path hbDir = CommonFSUtils.getRootDir(conf);
721    Path tableDir = CommonFSUtils.getTableDir(hbDir, tableName);
722    Path splitFile = new Path(tableDir, "_balancedSplit");
723    return new Pair<>(tableDir, splitFile);
724  }
725
726  static LinkedList<Pair<byte[], byte[]>> getSplits(final Connection connection,
727    TableName tableName, SplitAlgorithm splitAlgo) throws IOException {
728    Pair<Path, Path> tableDirAndSplitFile =
729      getTableDirAndSplitFile(connection.getConfiguration(), tableName);
730    Path tableDir = tableDirAndSplitFile.getFirst();
731    Path splitFile = tableDirAndSplitFile.getSecond();
732
733    FileSystem fs = tableDir.getFileSystem(connection.getConfiguration());
734
735    // Using strings because (new byte[]{0}).equals(new byte[]{0}) == false
736    Set<Pair<String, String>> daughterRegions = Sets.newHashSet();
737
738    // Does a split file exist?
739    if (!fs.exists(splitFile)) {
740      // NO = fresh start. calculate splits to make
741      LOG.debug("No " + splitFile.getName() + " file. Calculating splits ");
742
743      // Query meta for all regions in the table
744      Set<Pair<byte[], byte[]>> rows = Sets.newHashSet();
745      Pair<byte[][], byte[][]> tmp = null;
746      try (RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
747        tmp = regionLocator.getStartEndKeys();
748      }
749      Preconditions.checkArgument(tmp.getFirst().length == tmp.getSecond().length,
750        "Start and End rows should be equivalent");
751      for (int i = 0; i < tmp.getFirst().length; ++i) {
752        byte[] start = tmp.getFirst()[i], end = tmp.getSecond()[i];
753        if (start.length == 0) start = splitAlgo.firstRow();
754        if (end.length == 0) end = splitAlgo.lastRow();
755        rows.add(Pair.newPair(start, end));
756      }
757      LOG.debug("Table " + tableName + " has " + rows.size() + " regions that will be split.");
758
759      // prepare the split file
760      Path tmpFile = new Path(tableDir, "_balancedSplit_prepare");
761      FSDataOutputStream tmpOut = fs.create(tmpFile);
762
763      // calculate all the splits == [daughterRegions] = [(start, splitPoint)]
764      for (Pair<byte[], byte[]> r : rows) {
765        byte[] splitPoint = splitAlgo.split(r.getFirst(), r.getSecond());
766        String startStr = splitAlgo.rowToStr(r.getFirst());
767        String splitStr = splitAlgo.rowToStr(splitPoint);
768        daughterRegions.add(Pair.newPair(startStr, splitStr));
769        LOG.debug("Will Split [" + startStr + " , " + splitAlgo.rowToStr(r.getSecond()) + ") at "
770          + splitStr);
771        tmpOut.writeChars("+ " + startStr + splitAlgo.separator() + splitStr + "\n");
772      }
773      tmpOut.close();
774      fs.rename(tmpFile, splitFile);
775    } else {
776      LOG.debug("_balancedSplit file found. Replay log to restore state...");
777      RecoverLeaseFSUtils.recoverFileLease(fs, splitFile, connection.getConfiguration(), null);
778
779      // parse split file and process remaining splits
780      FSDataInputStream tmpIn = fs.open(splitFile);
781      StringBuilder sb = new StringBuilder(tmpIn.available());
782      while (tmpIn.available() > 0) {
783        sb.append(tmpIn.readChar());
784      }
785      tmpIn.close();
786      for (String line : sb.toString().split("\n")) {
787        String[] cmd = line.split(splitAlgo.separator());
788        Preconditions.checkArgument(3 == cmd.length);
789        byte[] start = splitAlgo.strToRow(cmd[1]);
790        String startStr = splitAlgo.rowToStr(start);
791        byte[] splitPoint = splitAlgo.strToRow(cmd[2]);
792        String splitStr = splitAlgo.rowToStr(splitPoint);
793        Pair<String, String> r = Pair.newPair(startStr, splitStr);
794        if (cmd[0].equals("+")) {
795          LOG.debug("Adding: " + r);
796          daughterRegions.add(r);
797        } else {
798          LOG.debug("Removing: " + r);
799          Preconditions.checkArgument(cmd[0].equals("-"), "Unknown option: " + cmd[0]);
800          Preconditions.checkState(daughterRegions.contains(r), "Missing row: " + r);
801          daughterRegions.remove(r);
802        }
803      }
804      LOG.debug("Done reading. " + daughterRegions.size() + " regions left.");
805    }
806    LinkedList<Pair<byte[], byte[]>> ret = Lists.newLinkedList();
807    for (Pair<String, String> r : daughterRegions) {
808      ret.add(Pair.newPair(splitAlgo.strToRow(r.getFirst()), splitAlgo.strToRow(r.getSecond())));
809    }
810    return ret;
811  }
812
813  /**
814   * HexStringSplit is a well-known {@link SplitAlgorithm} for choosing region boundaries. The
815   * format of a HexStringSplit region boundary is the ASCII representation of an MD5 checksum, or
816   * any other uniformly distributed hexadecimal value. Row are hex-encoded long values in the range
817   * <b>"00000000" =&gt; "FFFFFFFF"</b> and are left-padded with zeros to keep the same order
818   * lexicographically as if they were binary. Since this split algorithm uses hex strings as keys,
819   * it is easy to read &amp; write in the shell but takes up more space and may be non-intuitive.
820   */
821  public static class HexStringSplit extends NumberStringSplit {
822    final static String DEFAULT_MIN_HEX = "00000000";
823    final static String DEFAULT_MAX_HEX = "FFFFFFFF";
824    final static int RADIX_HEX = 16;
825
826    public HexStringSplit() {
827      super(DEFAULT_MIN_HEX, DEFAULT_MAX_HEX, RADIX_HEX);
828    }
829
830  }
831
832  /**
833   * The format of a DecimalStringSplit region boundary is the ASCII representation of reversed
834   * sequential number, or any other uniformly distributed decimal value. Row are decimal-encoded
835   * long values in the range <b>"00000000" =&gt; "99999999"</b> and are left-padded with zeros to
836   * keep the same order lexicographically as if they were binary.
837   */
838  public static class DecimalStringSplit extends NumberStringSplit {
839    final static String DEFAULT_MIN_DEC = "00000000";
840    final static String DEFAULT_MAX_DEC = "99999999";
841    final static int RADIX_DEC = 10;
842
843    public DecimalStringSplit() {
844      super(DEFAULT_MIN_DEC, DEFAULT_MAX_DEC, RADIX_DEC);
845    }
846
847  }
848
849  public abstract static class NumberStringSplit implements SplitAlgorithm {
850
851    String firstRow;
852    BigInteger firstRowInt;
853    String lastRow;
854    BigInteger lastRowInt;
855    int rowComparisonLength;
856    int radix;
857
858    NumberStringSplit(String minRow, String maxRow, int radix) {
859      this.firstRow = minRow;
860      this.lastRow = maxRow;
861      this.radix = radix;
862      this.firstRowInt = BigInteger.ZERO;
863      this.lastRowInt = new BigInteger(lastRow, this.radix);
864      this.rowComparisonLength = lastRow.length();
865    }
866
867    @Override
868    public byte[] split(byte[] start, byte[] end) {
869      BigInteger s = convertToBigInteger(start);
870      BigInteger e = convertToBigInteger(end);
871      Preconditions.checkArgument(!e.equals(BigInteger.ZERO));
872      return convertToByte(split2(s, e));
873    }
874
875    @Override
876    public byte[][] split(int n) {
877      Preconditions.checkArgument(lastRowInt.compareTo(firstRowInt) > 0,
878        "last row (%s) is configured less than first row (%s)", lastRow, firstRow);
879      // +1 to range because the last row is inclusive
880      BigInteger range = lastRowInt.subtract(firstRowInt).add(BigInteger.ONE);
881      Preconditions.checkState(range.compareTo(BigInteger.valueOf(n)) >= 0,
882        "split granularity (%s) is greater than the range (%s)", n, range);
883
884      BigInteger[] splits = new BigInteger[n - 1];
885      BigInteger sizeOfEachSplit = range.divide(BigInteger.valueOf(n));
886      for (int i = 1; i < n; i++) {
887        // NOTE: this means the last region gets all the slop.
888        // This is not a big deal if we're assuming n << MAXHEX
889        splits[i - 1] = firstRowInt.add(sizeOfEachSplit.multiply(BigInteger.valueOf(i)));
890      }
891      return convertToBytes(splits);
892    }
893
894    @Override
895    public byte[][] split(byte[] start, byte[] end, int numSplits, boolean inclusive) {
896      BigInteger s = convertToBigInteger(start);
897      BigInteger e = convertToBigInteger(end);
898
899      Preconditions.checkArgument(e.compareTo(s) > 0,
900        "last row (%s) is configured less than first row (%s)", rowToStr(end), end);
901      // +1 to range because the last row is inclusive
902      BigInteger range = e.subtract(s).add(BigInteger.ONE);
903      Preconditions.checkState(range.compareTo(BigInteger.valueOf(numSplits)) >= 0,
904        "split granularity (%s) is greater than the range (%s)", numSplits, range);
905
906      BigInteger[] splits = new BigInteger[numSplits - 1];
907      BigInteger sizeOfEachSplit = range.divide(BigInteger.valueOf(numSplits));
908      for (int i = 1; i < numSplits; i++) {
909        // NOTE: this means the last region gets all the slop.
910        // This is not a big deal if we're assuming n << MAXHEX
911        splits[i - 1] = s.add(sizeOfEachSplit.multiply(BigInteger.valueOf(i)));
912      }
913
914      if (inclusive) {
915        BigInteger[] inclusiveSplitPoints = new BigInteger[numSplits + 1];
916        inclusiveSplitPoints[0] = convertToBigInteger(start);
917        inclusiveSplitPoints[numSplits] = convertToBigInteger(end);
918        System.arraycopy(splits, 0, inclusiveSplitPoints, 1, splits.length);
919        return convertToBytes(inclusiveSplitPoints);
920      } else {
921        return convertToBytes(splits);
922      }
923    }
924
925    @Override
926    public byte[] firstRow() {
927      return convertToByte(firstRowInt);
928    }
929
930    @Override
931    public byte[] lastRow() {
932      return convertToByte(lastRowInt);
933    }
934
935    @Override
936    public void setFirstRow(String userInput) {
937      firstRow = userInput;
938      firstRowInt = new BigInteger(firstRow, radix);
939    }
940
941    @Override
942    public void setLastRow(String userInput) {
943      lastRow = userInput;
944      lastRowInt = new BigInteger(lastRow, radix);
945      // Precondition: lastRow > firstRow, so last's length is the greater
946      rowComparisonLength = lastRow.length();
947    }
948
949    @Override
950    public byte[] strToRow(String in) {
951      return convertToByte(new BigInteger(in, radix));
952    }
953
954    @Override
955    public String rowToStr(byte[] row) {
956      return Bytes.toStringBinary(row);
957    }
958
959    @Override
960    public String separator() {
961      return " ";
962    }
963
964    @Override
965    public void setFirstRow(byte[] userInput) {
966      firstRow = Bytes.toString(userInput);
967    }
968
969    @Override
970    public void setLastRow(byte[] userInput) {
971      lastRow = Bytes.toString(userInput);
972    }
973
974    /**
975     * Divide 2 numbers in half (for split algorithm)
976     * @param a number #1
977     * @param b number #2
978     * @return the midpoint of the 2 numbers
979     */
980    public BigInteger split2(BigInteger a, BigInteger b) {
981      return a.add(b).divide(BigInteger.valueOf(2)).abs();
982    }
983
984    /**
985     * Returns an array of bytes corresponding to an array of BigIntegers
986     * @param bigIntegers numbers to convert
987     * @return bytes corresponding to the bigIntegers
988     */
989    public byte[][] convertToBytes(BigInteger[] bigIntegers) {
990      byte[][] returnBytes = new byte[bigIntegers.length][];
991      for (int i = 0; i < bigIntegers.length; i++) {
992        returnBytes[i] = convertToByte(bigIntegers[i]);
993      }
994      return returnBytes;
995    }
996
997    /**
998     * Returns the bytes corresponding to the BigInteger
999     * @param bigInteger number to convert
1000     * @param pad        padding length
1001     * @return byte corresponding to input BigInteger
1002     */
1003    public byte[] convertToByte(BigInteger bigInteger, int pad) {
1004      String bigIntegerString = bigInteger.toString(radix);
1005      bigIntegerString = StringUtils.leftPad(bigIntegerString, pad, '0');
1006      return Bytes.toBytes(bigIntegerString);
1007    }
1008
1009    /**
1010     * Returns the bytes corresponding to the BigInteger
1011     * @param bigInteger number to convert
1012     * @return corresponding bytes
1013     */
1014    public byte[] convertToByte(BigInteger bigInteger) {
1015      return convertToByte(bigInteger, rowComparisonLength);
1016    }
1017
1018    /**
1019     * Returns the BigInteger represented by the byte array
1020     * @param row byte array representing row
1021     * @return the corresponding BigInteger
1022     */
1023    public BigInteger convertToBigInteger(byte[] row) {
1024      return (row.length > 0) ? new BigInteger(Bytes.toString(row), radix) : BigInteger.ZERO;
1025    }
1026
1027    @Override
1028    public String toString() {
1029      return this.getClass().getSimpleName() + " [" + rowToStr(firstRow()) + ","
1030        + rowToStr(lastRow()) + "]";
1031    }
1032  }
1033
1034  /**
1035   * A SplitAlgorithm that divides the space of possible keys evenly. Useful when the keys are
1036   * approximately uniform random bytes (e.g. hashes). Rows are raw byte values in the range <b>00
1037   * =&gt; FF</b> and are right-padded with zeros to keep the same memcmp() order. This is the
1038   * natural algorithm to use for a byte[] environment and saves space, but is not necessarily the
1039   * easiest for readability.
1040   */
1041  public static class UniformSplit implements SplitAlgorithm {
1042    static final byte xFF = (byte) 0xFF;
1043    byte[] firstRowBytes = ArrayUtils.EMPTY_BYTE_ARRAY;
1044    byte[] lastRowBytes = new byte[] { xFF, xFF, xFF, xFF, xFF, xFF, xFF, xFF };
1045
1046    @Override
1047    public byte[] split(byte[] start, byte[] end) {
1048      return Bytes.split(start, end, 1)[1];
1049    }
1050
1051    @Override
1052    public byte[][] split(int numRegions) {
1053      Preconditions.checkArgument(Bytes.compareTo(lastRowBytes, firstRowBytes) > 0,
1054        "last row (%s) is configured less than first row (%s)", Bytes.toStringBinary(lastRowBytes),
1055        Bytes.toStringBinary(firstRowBytes));
1056
1057      byte[][] splits = Bytes.split(firstRowBytes, lastRowBytes, true, numRegions - 1);
1058      Preconditions.checkState(splits != null,
1059        "Could not split region with given user input: " + this);
1060
1061      // remove endpoints, which are included in the splits list
1062
1063      return splits == null ? null : Arrays.copyOfRange(splits, 1, splits.length - 1);
1064    }
1065
1066    @Override
1067    public byte[][] split(byte[] start, byte[] end, int numSplits, boolean inclusive) {
1068      if (Arrays.equals(start, HConstants.EMPTY_BYTE_ARRAY)) {
1069        start = firstRowBytes;
1070      }
1071      if (Arrays.equals(end, HConstants.EMPTY_BYTE_ARRAY)) {
1072        end = lastRowBytes;
1073      }
1074      Preconditions.checkArgument(Bytes.compareTo(end, start) > 0,
1075        "last row (%s) is configured less than first row (%s)", Bytes.toStringBinary(end),
1076        Bytes.toStringBinary(start));
1077
1078      byte[][] splits = Bytes.split(start, end, true, numSplits - 1);
1079      Preconditions.checkState(splits != null,
1080        "Could not calculate input splits with given user input: " + this);
1081      if (inclusive) {
1082        return splits;
1083      } else {
1084        // remove endpoints, which are included in the splits list
1085        return Arrays.copyOfRange(splits, 1, splits.length - 1);
1086      }
1087    }
1088
1089    @Override
1090    public byte[] firstRow() {
1091      return firstRowBytes;
1092    }
1093
1094    @Override
1095    public byte[] lastRow() {
1096      return lastRowBytes;
1097    }
1098
1099    @Override
1100    public void setFirstRow(String userInput) {
1101      firstRowBytes = Bytes.toBytesBinary(userInput);
1102    }
1103
1104    @Override
1105    public void setLastRow(String userInput) {
1106      lastRowBytes = Bytes.toBytesBinary(userInput);
1107    }
1108
1109    @Override
1110    public void setFirstRow(byte[] userInput) {
1111      firstRowBytes = userInput;
1112    }
1113
1114    @Override
1115    public void setLastRow(byte[] userInput) {
1116      lastRowBytes = userInput;
1117    }
1118
1119    @Override
1120    public byte[] strToRow(String input) {
1121      return Bytes.toBytesBinary(input);
1122    }
1123
1124    @Override
1125    public String rowToStr(byte[] row) {
1126      return Bytes.toStringBinary(row);
1127    }
1128
1129    @Override
1130    public String separator() {
1131      return ",";
1132    }
1133
1134    @Override
1135    public String toString() {
1136      return this.getClass().getSimpleName() + " [" + rowToStr(firstRow()) + ","
1137        + rowToStr(lastRow()) + "]";
1138    }
1139  }
1140}