001/*
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.master;
020
021import java.io.IOException;
022import java.io.InputStream;
023import java.util.ArrayList;
024import java.util.Comparator;
025import java.util.HashSet;
026import java.util.List;
027import java.util.Map;
028import java.util.Properties;
029import java.util.TreeMap;
030import java.util.concurrent.atomic.AtomicBoolean;
031import java.util.stream.Collectors;
032
033import org.apache.hadoop.conf.Configuration;
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.MetaTableAccessor;
039import org.apache.hadoop.hbase.RegionLocations;
040import org.apache.hadoop.hbase.ScheduledChore;
041import org.apache.hadoop.hbase.ServerName;
042import org.apache.hadoop.hbase.TableName;
043import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
044import org.apache.hadoop.hbase.client.Connection;
045import org.apache.hadoop.hbase.client.ConnectionFactory;
046import org.apache.hadoop.hbase.client.Get;
047import org.apache.hadoop.hbase.client.Put;
048import org.apache.hadoop.hbase.client.RegionInfo;
049import org.apache.hadoop.hbase.client.Result;
050import org.apache.hadoop.hbase.client.Table;
051import org.apache.hadoop.hbase.client.TableDescriptor;
052import org.apache.hadoop.hbase.client.TableState;
053import org.apache.hadoop.hbase.master.assignment.AssignmentManager;
054import org.apache.hadoop.hbase.master.assignment.GCMultipleMergedRegionsProcedure;
055import org.apache.hadoop.hbase.master.assignment.GCRegionProcedure;
056import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
057import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
058import org.apache.hadoop.hbase.regionserver.HRegionFileSystem;
059import org.apache.hadoop.hbase.util.Bytes;
060import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
061import org.apache.hadoop.hbase.util.FSUtils;
062import org.apache.hadoop.hbase.util.Pair;
063import org.apache.hadoop.hbase.util.PairOfSameType;
064import org.apache.hadoop.hbase.util.Threads;
065import org.apache.yetus.audience.InterfaceAudience;
066import org.slf4j.Logger;
067import org.slf4j.LoggerFactory;
068
069import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
070
071/**
072 * A janitor for the catalog tables. Scans the <code>hbase:meta</code> catalog
073 * table on a period. Makes a lastReport on state of hbase:meta. Looks for unused
074 * regions to garbage collect. Scan of hbase:meta runs if we are NOT in maintenance
075 * mode, if we are NOT shutting down, AND if the assignmentmanager is loaded.
076 * Playing it safe, we will garbage collect no-longer needed region references
077 * only if there are no regions-in-transition (RIT).
078 */
079// TODO: Only works with single hbase:meta region currently.  Fix.
080// TODO: Should it start over every time? Could it continue if runs into problem? Only if
081// problem does not mess up 'results'.
082@InterfaceAudience.Private
083public class CatalogJanitor extends ScheduledChore {
084  private static final Logger LOG = LoggerFactory.getLogger(CatalogJanitor.class.getName());
085  private final AtomicBoolean alreadyRunning = new AtomicBoolean(false);
086  private final AtomicBoolean enabled = new AtomicBoolean(true);
087  private final MasterServices services;
088
089  /**
090   * Saved report from last hbase:meta scan to completion. May be stale if having trouble
091   * completing scan. Check its date.
092   */
093  private volatile Report lastReport;
094
095  CatalogJanitor(final MasterServices services) {
096    super("CatalogJanitor-" + services.getServerName().toShortString(), services,
097      services.getConfiguration().getInt("hbase.catalogjanitor.interval", 300000));
098    this.services = services;
099  }
100
101  @Override
102  protected boolean initialChore() {
103    try {
104      if (getEnabled()) {
105        scan();
106      }
107    } catch (IOException e) {
108      LOG.warn("Failed initial janitorial scan of hbase:meta table", e);
109      return false;
110    }
111    return true;
112  }
113
114  boolean setEnabled(final boolean enabled) {
115    boolean alreadyEnabled = this.enabled.getAndSet(enabled);
116    // If disabling is requested on an already enabled chore, we could have an active
117    // scan still going on, callers might not be aware of that and do further action thinkng
118    // that no action would be from this chore.  In this case, the right action is to wait for
119    // the active scan to complete before exiting this function.
120    if (!enabled && alreadyEnabled) {
121      while (alreadyRunning.get()) {
122        Threads.sleepWithoutInterrupt(100);
123      }
124    }
125    return alreadyEnabled;
126  }
127
128  boolean getEnabled() {
129    return this.enabled.get();
130  }
131
132  @Override
133  protected void chore() {
134    try {
135      AssignmentManager am = this.services.getAssignmentManager();
136      if (getEnabled() && !this.services.isInMaintenanceMode() &&
137          !this.services.getServerManager().isClusterShutdown() &&
138          isMetaLoaded(am)) {
139        scan();
140      } else {
141        LOG.warn("CatalogJanitor is disabled! Enabled=" + getEnabled() + 
142          ", maintenanceMode=" + this.services.isInMaintenanceMode() + ", am=" + am +
143          ", metaLoaded=" + isMetaLoaded(am) + ", hasRIT=" + isRIT(am) +
144          " clusterShutDown=" + this.services.getServerManager().isClusterShutdown());
145      }
146    } catch (IOException e) {
147      LOG.warn("Failed janitorial scan of hbase:meta table", e);
148    }
149  }
150
151  private static boolean isMetaLoaded(AssignmentManager am) {
152    return am != null && am.isMetaLoaded();
153  }
154
155  private static boolean isRIT(AssignmentManager am) {
156    return isMetaLoaded(am) && am.hasRegionsInTransition();
157  }
158
159  /**
160   * Run janitorial scan of catalog <code>hbase:meta</code> table looking for
161   * garbage to collect.
162   * @return How many items gc'd whether for merge or split.
163   */
164  int scan() throws IOException {
165    int gcs = 0;
166    try {
167      if (!alreadyRunning.compareAndSet(false, true)) {
168        LOG.debug("CatalogJanitor already running");
169        return gcs;
170      }
171      Report report = scanForReport();
172      this.lastReport = report;
173      if (!report.isEmpty()) {
174        LOG.warn(report.toString());
175      }
176
177      if (isRIT(this.services.getAssignmentManager())) {
178        LOG.warn("Playing-it-safe skipping merge/split gc'ing of regions from hbase:meta while " +
179            "regions-in-transition (RIT)");
180      }
181      Map<RegionInfo, Result> mergedRegions = report.mergedRegions;
182      for (Map.Entry<RegionInfo, Result> e : mergedRegions.entrySet()) {
183        if (this.services.isInMaintenanceMode()) {
184          // Stop cleaning if the master is in maintenance mode
185          break;
186        }
187
188        List<RegionInfo> parents = MetaTableAccessor.getMergeRegions(e.getValue().rawCells());
189        if (parents != null && cleanMergeRegion(e.getKey(), parents)) {
190          gcs++;
191        }
192      }
193      // Clean split parents
194      Map<RegionInfo, Result> splitParents = report.splitParents;
195
196      // Now work on our list of found parents. See if any we can clean up.
197      HashSet<String> parentNotCleaned = new HashSet<>();
198      for (Map.Entry<RegionInfo, Result> e : splitParents.entrySet()) {
199        if (this.services.isInMaintenanceMode()) {
200          // Stop cleaning if the master is in maintenance mode
201          break;
202        }
203
204        if (!parentNotCleaned.contains(e.getKey().getEncodedName()) &&
205            cleanParent(e.getKey(), e.getValue())) {
206          gcs++;
207        } else {
208          // We could not clean the parent, so it's daughters should not be
209          // cleaned either (HBASE-6160)
210          PairOfSameType<RegionInfo> daughters =
211              MetaTableAccessor.getDaughterRegions(e.getValue());
212          parentNotCleaned.add(daughters.getFirst().getEncodedName());
213          parentNotCleaned.add(daughters.getSecond().getEncodedName());
214        }
215      }
216      return gcs;
217    } finally {
218      alreadyRunning.set(false);
219    }
220  }
221
222  /**
223   * Scan hbase:meta.
224   * @return Return generated {@link Report}
225   */
226  Report scanForReport() throws IOException {
227    ReportMakingVisitor visitor = new ReportMakingVisitor(this.services);
228    // Null tablename means scan all of meta.
229    MetaTableAccessor.scanMetaForTableRegions(this.services.getConnection(), visitor, null);
230    return visitor.getReport();
231  }
232
233  /**
234   * @return Returns last published Report that comes of last successful scan
235   *   of hbase:meta.
236   */
237  public Report getLastReport() {
238    return this.lastReport;
239  }
240
241  /**
242   * If merged region no longer holds reference to the merge regions, archive
243   * merge region on hdfs and perform deleting references in hbase:meta
244   * @return true if we delete references in merged region on hbase:meta and archive
245   *   the files on the file system
246   */
247  private boolean cleanMergeRegion(final RegionInfo mergedRegion, List<RegionInfo> parents)
248      throws IOException {
249    FileSystem fs = this.services.getMasterFileSystem().getFileSystem();
250    Path rootdir = this.services.getMasterFileSystem().getRootDir();
251    Path tabledir = FSUtils.getTableDir(rootdir, mergedRegion.getTable());
252    TableDescriptor htd = getDescriptor(mergedRegion.getTable());
253    HRegionFileSystem regionFs = null;
254    try {
255      regionFs = HRegionFileSystem.openRegionFromFileSystem(
256          this.services.getConfiguration(), fs, tabledir, mergedRegion, true);
257    } catch (IOException e) {
258      LOG.warn("Merged region does not exist: " + mergedRegion.getEncodedName());
259    }
260    if (regionFs == null || !regionFs.hasReferences(htd)) {
261      LOG.debug("Deleting parents ({}) from fs; merged child {} no longer holds references",
262           parents.stream().map(r -> RegionInfo.getShortNameToLog(r)).
263              collect(Collectors.joining(", ")),
264          mergedRegion);
265      ProcedureExecutor<MasterProcedureEnv> pe = this.services.getMasterProcedureExecutor();
266      pe.submitProcedure(new GCMultipleMergedRegionsProcedure(pe.getEnvironment(),
267          mergedRegion,  parents));
268      for (RegionInfo ri:  parents) {
269        // The above scheduled GCMultipleMergedRegionsProcedure does the below.
270        // Do we need this?
271        this.services.getAssignmentManager().getRegionStates().deleteRegion(ri);
272        this.services.getServerManager().removeRegion(ri);
273      }
274      return true;
275    }
276    return false;
277  }
278
279  /**
280   * Compare HRegionInfos in a way that has split parents sort BEFORE their daughters.
281   */
282  static class SplitParentFirstComparator implements Comparator<RegionInfo> {
283    Comparator<byte[]> rowEndKeyComparator = new Bytes.RowEndKeyComparator();
284    @Override
285    public int compare(RegionInfo left, RegionInfo right) {
286      // This comparator differs from the one RegionInfo in that it sorts
287      // parent before daughters.
288      if (left == null) {
289        return -1;
290      }
291      if (right == null) {
292        return 1;
293      }
294      // Same table name.
295      int result = left.getTable().compareTo(right.getTable());
296      if (result != 0) {
297        return result;
298      }
299      // Compare start keys.
300      result = Bytes.compareTo(left.getStartKey(), right.getStartKey());
301      if (result != 0) {
302        return result;
303      }
304      // Compare end keys, but flip the operands so parent comes first
305      result = rowEndKeyComparator.compare(right.getEndKey(), left.getEndKey());
306
307      return result;
308    }
309  }
310
311  /**
312   * If daughters no longer hold reference to the parents, delete the parent.
313   * @param parent RegionInfo of split offlined parent
314   * @param rowContent Content of <code>parent</code> row in
315   * <code>metaRegionName</code>
316   * @return True if we removed <code>parent</code> from meta table and from
317   * the filesystem.
318   */
319  boolean cleanParent(final RegionInfo parent, Result rowContent)
320  throws IOException {
321    // Check whether it is a merged region and if it is clean of references.
322    if (MetaTableAccessor.hasMergeRegions(rowContent.rawCells())) {
323      // Wait until clean of merge parent regions first
324      return false;
325    }
326    // Run checks on each daughter split.
327    PairOfSameType<RegionInfo> daughters = MetaTableAccessor.getDaughterRegions(rowContent);
328    Pair<Boolean, Boolean> a = checkDaughterInFs(parent, daughters.getFirst());
329    Pair<Boolean, Boolean> b = checkDaughterInFs(parent, daughters.getSecond());
330    if (hasNoReferences(a) && hasNoReferences(b)) {
331      String daughterA = daughters.getFirst() != null?
332          daughters.getFirst().getShortNameToLog(): "null";
333      String daughterB = daughters.getSecond() != null?
334          daughters.getSecond().getShortNameToLog(): "null";
335      LOG.debug("Deleting region " + parent.getShortNameToLog() +
336        " because daughters -- " + daughterA + ", " + daughterB +
337        " -- no longer hold references");
338      ProcedureExecutor<MasterProcedureEnv> pe = this.services.getMasterProcedureExecutor();
339      pe.submitProcedure(new GCRegionProcedure(pe.getEnvironment(), parent));
340      // Remove from in-memory states
341      this.services.getAssignmentManager().getRegionStates().deleteRegion(parent);
342      this.services.getServerManager().removeRegion(parent);
343      return true;
344    }
345    return false;
346  }
347
348  /**
349   * @param p A pair where the first boolean says whether or not the daughter
350   * region directory exists in the filesystem and then the second boolean says
351   * whether the daughter has references to the parent.
352   * @return True the passed <code>p</code> signifies no references.
353   */
354  private boolean hasNoReferences(final Pair<Boolean, Boolean> p) {
355    return !p.getFirst() || !p.getSecond();
356  }
357
358  /**
359   * Checks if a daughter region -- either splitA or splitB -- still holds
360   * references to parent.
361   * @param parent Parent region
362   * @param daughter Daughter region
363   * @return A pair where the first boolean says whether or not the daughter
364   *   region directory exists in the filesystem and then the second boolean says
365   *   whether the daughter has references to the parent.
366   */
367  private Pair<Boolean, Boolean> checkDaughterInFs(final RegionInfo parent,
368    final RegionInfo daughter)
369  throws IOException {
370    if (daughter == null)  {
371      return new Pair<>(Boolean.FALSE, Boolean.FALSE);
372    }
373
374    FileSystem fs = this.services.getMasterFileSystem().getFileSystem();
375    Path rootdir = this.services.getMasterFileSystem().getRootDir();
376    Path tabledir = FSUtils.getTableDir(rootdir, daughter.getTable());
377
378    Path daughterRegionDir = new Path(tabledir, daughter.getEncodedName());
379
380    HRegionFileSystem regionFs;
381
382    try {
383      if (!FSUtils.isExists(fs, daughterRegionDir)) {
384        return new Pair<>(Boolean.FALSE, Boolean.FALSE);
385      }
386    } catch (IOException ioe) {
387      LOG.error("Error trying to determine if daughter region exists, " +
388               "assuming exists and has references", ioe);
389      return new Pair<>(Boolean.TRUE, Boolean.TRUE);
390    }
391
392    boolean references = false;
393    TableDescriptor parentDescriptor = getDescriptor(parent.getTable());
394    try {
395      regionFs = HRegionFileSystem.openRegionFromFileSystem(
396          this.services.getConfiguration(), fs, tabledir, daughter, true);
397
398      for (ColumnFamilyDescriptor family: parentDescriptor.getColumnFamilies()) {
399        if ((references = regionFs.hasReferences(family.getNameAsString()))) {
400          break;
401        }
402      }
403    } catch (IOException e) {
404      LOG.error("Error trying to determine referenced files from : " + daughter.getEncodedName()
405          + ", to: " + parent.getEncodedName() + " assuming has references", e);
406      return new Pair<>(Boolean.TRUE, Boolean.TRUE);
407    }
408    return new Pair<>(Boolean.TRUE, references);
409  }
410
411  private TableDescriptor getDescriptor(final TableName tableName) throws IOException {
412    return this.services.getTableDescriptors().get(tableName);
413  }
414
415  /**
416   * Checks if the specified region has merge qualifiers, if so, try to clean them.
417   * @return true if no info:merge* columns; i.e. the specified region doesn't have
418   *   any merge qualifiers.
419   */
420  public boolean cleanMergeQualifier(final RegionInfo region) throws IOException {
421    // Get merge regions if it is a merged region and already has merge qualifier
422    List<RegionInfo> parents = MetaTableAccessor.getMergeRegions(this.services.getConnection(),
423        region.getRegionName());
424    if (parents == null || parents.isEmpty()) {
425      // It doesn't have merge qualifier, no need to clean
426      return true;
427    }
428    return cleanMergeRegion(region, parents);
429  }
430
431  /**
432   * Report made by ReportMakingVisitor
433   */
434  public static class Report {
435    private final long now = EnvironmentEdgeManager.currentTime();
436
437    // Keep Map of found split parents. These are candidates for cleanup.
438    // Use a comparator that has split parents come before its daughters.
439    final Map<RegionInfo, Result> splitParents = new TreeMap<>(new SplitParentFirstComparator());
440    final Map<RegionInfo, Result> mergedRegions = new TreeMap<>(RegionInfo.COMPARATOR);
441    int count = 0;
442
443    private final List<Pair<RegionInfo, RegionInfo>> holes = new ArrayList<>();
444    private final List<Pair<RegionInfo, RegionInfo>> overlaps = new ArrayList<>();
445    private final List<Pair<RegionInfo, ServerName>> unknownServers = new ArrayList<>();
446    private final List<byte []> emptyRegionInfo = new ArrayList<>();
447
448    @VisibleForTesting
449    Report() {}
450
451    public long getCreateTime() {
452      return this.now;
453    }
454
455    public List<Pair<RegionInfo, RegionInfo>> getHoles() {
456      return this.holes;
457    }
458
459    public List<Pair<RegionInfo, RegionInfo>> getOverlaps() {
460      return this.overlaps;
461    }
462
463    public List<Pair<RegionInfo, ServerName>> getUnknownServers() {
464      return unknownServers;
465    }
466
467    public List<byte[]> getEmptyRegionInfo() {
468      return emptyRegionInfo;
469    }
470
471    /**
472     * @return True if an 'empty' lastReport -- no problems found.
473     */
474    public boolean isEmpty() {
475      return this.holes.isEmpty() && this.overlaps.isEmpty() && this.unknownServers.isEmpty() &&
476          this.emptyRegionInfo.isEmpty();
477    }
478
479    @Override
480    public String toString() {
481      StringBuffer sb = new StringBuffer();
482      for (Pair<RegionInfo, RegionInfo> p: this.holes) {
483        if (sb.length() > 0) {
484          sb.append(", ");
485        }
486        sb.append("hole=" + p.getFirst().getRegionNameAsString() + "/" +
487            p.getSecond().getRegionNameAsString());
488      }
489      for (Pair<RegionInfo, RegionInfo> p: this.overlaps) {
490        if (sb.length() > 0) {
491          sb.append(", ");
492        }
493        sb.append("overlap=" + p.getFirst().getRegionNameAsString() + "/" +
494            p.getSecond().getRegionNameAsString());
495      }
496      for (byte [] r: this.emptyRegionInfo) {
497        if (sb.length() > 0) {
498          sb.append(", ");
499        }
500        sb.append("empty=").append(Bytes.toStringBinary(r));
501      }
502      for (Pair<RegionInfo, ServerName> p: this.unknownServers) {
503        if (sb.length() > 0) {
504          sb.append(", ");
505        }
506        sb.append("unknown_server=").append(p.getSecond()).append("/").
507            append(p.getFirst().getRegionNameAsString());
508      }
509      return sb.toString();
510    }
511  }
512
513  /**
514   * Visitor we use in here in CatalogJanitor to go against hbase:meta table.
515   * Generates a Report made of a collection of split parents and counts of rows
516   * in the hbase:meta table. Also runs hbase:meta consistency checks to
517   * generate more report. Report is NOT ready until after this visitor has been
518   * {@link #close()}'d.
519   */
520  static class ReportMakingVisitor implements MetaTableAccessor.CloseableVisitor {
521    private final MasterServices services;
522    private volatile boolean closed;
523
524    /**
525     * Report is not done until after the close has been called.
526     * @see #close()
527     * @see #getReport()
528     */
529    private Report report = new Report();
530
531    /**
532     * RegionInfo from previous row.
533     */
534    private RegionInfo previous = null;
535
536    ReportMakingVisitor(MasterServices services) {
537      this.services = services;
538    }
539
540    /**
541     * Do not call until after {@link #close()}.
542     * Will throw a {@link RuntimeException} if you do.
543     */
544    Report getReport() {
545      if (!this.closed) {
546        throw new RuntimeException("Report not ready until after close()");
547      }
548      return this.report;
549    }
550
551    @Override
552    public boolean visit(Result r) {
553      if (r == null || r.isEmpty()) {
554        return true;
555      }
556      this.report.count++;
557      RegionInfo regionInfo = metaTableConsistencyCheck(r);
558      if (regionInfo != null) {
559        LOG.trace(regionInfo.toString());
560        if (regionInfo.isSplitParent()) { // splitParent means split and offline.
561          this.report.splitParents.put(regionInfo, r);
562        }
563        if (MetaTableAccessor.hasMergeRegions(r.rawCells())) {
564          this.report.mergedRegions.put(regionInfo, r);
565        }
566      }
567      // Returning true means "keep scanning"
568      return true;
569    }
570
571    /**
572     * Check row.
573     * @param metaTableRow Row from hbase:meta table.
574     * @return Returns default regioninfo found in row parse as a convenience to save
575     *   on having to do a double-parse of Result.
576     */
577    private RegionInfo metaTableConsistencyCheck(Result metaTableRow) {
578      // Locations comes back null if the RegionInfo field is empty.
579      // If locations is null, ensure the regioninfo is for sure empty before progressing.
580      // If really empty, report as missing regioninfo!  Otherwise, can run server check
581      // and get RegionInfo from locations.
582      RegionLocations locations = MetaTableAccessor.getRegionLocations(metaTableRow);
583      RegionInfo ri = (locations == null)?
584          MetaTableAccessor.getRegionInfo(metaTableRow, MetaTableAccessor.getRegionInfoColumn()):
585          locations.getDefaultRegionLocation().getRegion();
586
587      if (ri == null) {
588        this.report.emptyRegionInfo.add(metaTableRow.getRow());
589        return ri;
590      }
591
592      if (!Bytes.equals(metaTableRow.getRow(), ri.getRegionName())) {
593        LOG.warn("INCONSISTENCY: Row name is not equal to serialized info:regioninfo content; " +
594                "row={} {}; See if RegionInfo is referenced in another hbase:meta row? Delete?",
595            Bytes.toStringBinary(metaTableRow.getRow()), ri.getRegionNameAsString());
596        return null;
597      }
598      // If table is disabled, skip integrity check.
599      if (!isTableDisabled(ri)) {
600        if (isTableTransition(ri)) {
601          // On table transition, look to see if last region was last in table
602          // and if this is the first. Report 'hole' if neither is true.
603          // HBCK1 used to have a special category for missing start or end keys.
604          // We'll just lump them in as 'holes'.
605          if ((this.previous != null && !this.previous.isLast()) || !ri.isFirst()) {
606            addHole(this.previous == null? RegionInfo.UNDEFINED: this.previous, ri);
607          }
608        } else {
609          if (!this.previous.isNext(ri)) {
610            if (this.previous.isOverlap(ri)) {
611              addOverlap(this.previous, ri);
612            } else {
613              addHole(this.previous, ri);
614            }
615          }
616        }
617      }
618      this.previous = ri;
619      return ri;
620    }
621
622    private void addOverlap(RegionInfo a, RegionInfo b) {
623      this.report.overlaps.add(new Pair<>(a, b));
624    }
625
626    private void addHole(RegionInfo a, RegionInfo b) {
627      this.report.holes.add(new Pair<>(a, b));
628    }
629
630    /**
631     * @return True if table is disabled or disabling; defaults false!
632     */
633    boolean isTableDisabled(RegionInfo ri) {
634      if (ri == null) {
635        return false;
636      }
637      if (this.services == null) {
638        return false;
639      }
640      if (this.services.getTableStateManager() == null) {
641        return false;
642      }
643      TableState state = null;
644      try {
645        state = this.services.getTableStateManager().getTableState(ri.getTable());
646      } catch (IOException e) {
647        LOG.warn("Failed getting table state", e);
648      }
649      return state != null && state.isDisabledOrDisabling();
650    }
651
652    /**
653     * @return True iff first row in hbase:meta or if we've broached a new table in hbase:meta
654     */
655    private boolean isTableTransition(RegionInfo ri) {
656      return this.previous == null ||
657          !this.previous.getTable().equals(ri.getTable());
658    }
659
660    @Override
661    public void close() throws IOException {
662      // This is a table transition... after the last region. Check previous.
663      // Should be last region. If not, its a hole on end of laster table.
664      if (this.previous != null && !this.previous.isLast()) {
665        addHole(this.previous, RegionInfo.UNDEFINED);
666      }
667      this.closed = true;
668    }
669  }
670
671  private static void checkLog4jProperties() {
672    String filename = "log4j.properties";
673    try {
674      final InputStream inStream =
675          CatalogJanitor.class.getClassLoader().getResourceAsStream(filename);
676      if (inStream != null) {
677        new Properties().load(inStream);
678      } else {
679        System.out.println("No " + filename + " on classpath; Add one else no logging output!");
680      }
681    } catch (IOException e) {
682      LOG.error("Log4j check failed", e);
683    }
684  }
685
686  /**
687   * For testing against a cluster.
688   * Doesn't have a MasterServices context so does not report on good vs bad servers.
689   */
690  public static void main(String [] args) throws IOException {
691    checkLog4jProperties();
692    ReportMakingVisitor visitor = new ReportMakingVisitor(null);
693    Configuration configuration = HBaseConfiguration.create();
694    configuration.setBoolean("hbase.defaults.for.version.skip", true);
695    try (Connection connection = ConnectionFactory.createConnection(configuration)) {
696      /* Used to generate an overlap.
697      */
698      Get g = new Get(Bytes.toBytes("t2,40,1564119846424.1db8c57d64e0733e0f027aaeae7a0bf0."));
699      g.addColumn(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER);
700      try (Table t = connection.getTable(TableName.META_TABLE_NAME)) {
701        Result r = t.get(g);
702        byte [] row = g.getRow();
703        row[row.length - 2] <<= row[row.length - 2];
704        Put p = new Put(g.getRow());
705        p.addColumn(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER,
706            r.getValue(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER));
707        t.put(p);
708      }
709      MetaTableAccessor.scanMetaForTableRegions(connection, visitor, null);
710      Report report = visitor.getReport();
711      LOG.info(report != null? report.toString(): "empty");
712    }
713  }
714}