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.Closeable; 021import java.io.FileNotFoundException; 022import java.io.IOException; 023import java.io.InterruptedIOException; 024import java.io.PrintWriter; 025import java.io.StringWriter; 026import java.net.InetAddress; 027import java.net.URI; 028import java.util.ArrayList; 029import java.util.Collection; 030import java.util.Collections; 031import java.util.Comparator; 032import java.util.EnumSet; 033import java.util.HashMap; 034import java.util.HashSet; 035import java.util.Iterator; 036import java.util.List; 037import java.util.Locale; 038import java.util.Map; 039import java.util.Map.Entry; 040import java.util.Objects; 041import java.util.Optional; 042import java.util.Set; 043import java.util.SortedMap; 044import java.util.TreeMap; 045import java.util.Vector; 046import java.util.concurrent.Callable; 047import java.util.concurrent.ConcurrentSkipListMap; 048import java.util.concurrent.ExecutionException; 049import java.util.concurrent.ExecutorService; 050import java.util.concurrent.Executors; 051import java.util.concurrent.Future; 052import java.util.concurrent.FutureTask; 053import java.util.concurrent.ScheduledThreadPoolExecutor; 054import java.util.concurrent.TimeUnit; 055import java.util.concurrent.TimeoutException; 056import java.util.concurrent.atomic.AtomicBoolean; 057import java.util.concurrent.atomic.AtomicInteger; 058import java.util.stream.Collectors; 059 060import org.apache.commons.io.IOUtils; 061import org.apache.commons.lang3.StringUtils; 062import org.apache.hadoop.conf.Configuration; 063import org.apache.hadoop.conf.Configured; 064import org.apache.hadoop.fs.FSDataOutputStream; 065import org.apache.hadoop.fs.FileStatus; 066import org.apache.hadoop.fs.FileSystem; 067import org.apache.hadoop.fs.Path; 068import org.apache.hadoop.fs.permission.FsAction; 069import org.apache.hadoop.fs.permission.FsPermission; 070import org.apache.hadoop.hbase.Abortable; 071import org.apache.hadoop.hbase.Cell; 072import org.apache.hadoop.hbase.CellUtil; 073import org.apache.hadoop.hbase.ClusterMetrics; 074import org.apache.hadoop.hbase.ClusterMetrics.Option; 075import org.apache.hadoop.hbase.HBaseConfiguration; 076import org.apache.hadoop.hbase.HBaseInterfaceAudience; 077import org.apache.hadoop.hbase.HConstants; 078import org.apache.hadoop.hbase.HRegionLocation; 079import org.apache.hadoop.hbase.KeyValue; 080import org.apache.hadoop.hbase.MasterNotRunningException; 081import org.apache.hadoop.hbase.MetaTableAccessor; 082import org.apache.hadoop.hbase.RegionLocations; 083import org.apache.hadoop.hbase.ServerName; 084import org.apache.hadoop.hbase.TableName; 085import org.apache.hadoop.hbase.TableNotFoundException; 086import org.apache.hadoop.hbase.ZooKeeperConnectionException; 087import org.apache.hadoop.hbase.client.Admin; 088import org.apache.hadoop.hbase.client.ClusterConnection; 089import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor; 090import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder; 091import org.apache.hadoop.hbase.client.Connection; 092import org.apache.hadoop.hbase.client.ConnectionFactory; 093import org.apache.hadoop.hbase.client.Delete; 094import org.apache.hadoop.hbase.client.Get; 095import org.apache.hadoop.hbase.client.Put; 096import org.apache.hadoop.hbase.client.RegionInfo; 097import org.apache.hadoop.hbase.client.RegionInfoBuilder; 098import org.apache.hadoop.hbase.client.RegionReplicaUtil; 099import org.apache.hadoop.hbase.client.Result; 100import org.apache.hadoop.hbase.client.ResultScanner; 101import org.apache.hadoop.hbase.client.RowMutations; 102import org.apache.hadoop.hbase.client.Scan; 103import org.apache.hadoop.hbase.client.Table; 104import org.apache.hadoop.hbase.client.TableDescriptor; 105import org.apache.hadoop.hbase.client.TableDescriptorBuilder; 106import org.apache.hadoop.hbase.client.TableState; 107import org.apache.hadoop.hbase.io.FileLink; 108import org.apache.hadoop.hbase.io.HFileLink; 109import org.apache.hadoop.hbase.io.hfile.CacheConfig; 110import org.apache.hadoop.hbase.io.hfile.HFile; 111import org.apache.hadoop.hbase.master.RegionState; 112import org.apache.hadoop.hbase.regionserver.HRegion; 113import org.apache.hadoop.hbase.regionserver.HRegionFileSystem; 114import org.apache.hadoop.hbase.regionserver.StoreFileInfo; 115import org.apache.hadoop.hbase.replication.ReplicationException; 116import org.apache.hadoop.hbase.replication.ReplicationPeerDescription; 117import org.apache.hadoop.hbase.replication.ReplicationQueueStorage; 118import org.apache.hadoop.hbase.replication.ReplicationStorageFactory; 119import org.apache.hadoop.hbase.security.UserProvider; 120import org.apache.hadoop.hbase.util.Bytes.ByteArrayComparator; 121import org.apache.hadoop.hbase.util.HbckErrorReporter.ERROR_CODE; 122import org.apache.hadoop.hbase.util.hbck.HFileCorruptionChecker; 123import org.apache.hadoop.hbase.util.hbck.ReplicationChecker; 124import org.apache.hadoop.hbase.util.hbck.TableIntegrityErrorHandler; 125import org.apache.hadoop.hbase.wal.WALSplitUtil; 126import org.apache.hadoop.hbase.zookeeper.ZKUtil; 127import org.apache.hadoop.hbase.zookeeper.ZKWatcher; 128import org.apache.hadoop.hbase.zookeeper.ZNodePaths; 129import org.apache.hadoop.hdfs.protocol.AlreadyBeingCreatedException; 130import org.apache.hadoop.ipc.RemoteException; 131import org.apache.hadoop.security.AccessControlException; 132import org.apache.hadoop.security.UserGroupInformation; 133import org.apache.hadoop.util.ReflectionUtils; 134import org.apache.hadoop.util.Tool; 135import org.apache.hadoop.util.ToolRunner; 136import org.apache.yetus.audience.InterfaceAudience; 137import org.apache.yetus.audience.InterfaceStability; 138import org.apache.zookeeper.KeeperException; 139import org.slf4j.Logger; 140import org.slf4j.LoggerFactory; 141 142import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting; 143import org.apache.hbase.thirdparty.com.google.common.base.Joiner; 144import org.apache.hbase.thirdparty.com.google.common.base.Preconditions; 145import org.apache.hbase.thirdparty.com.google.common.collect.Lists; 146import org.apache.hbase.thirdparty.com.google.common.collect.Sets; 147 148import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 149import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.AdminService.BlockingInterface; 150 151/** 152 * HBaseFsck (hbck) is a tool for checking and repairing region consistency and 153 * table integrity problems in a corrupted HBase. This tool was written for hbase-1.x. It does not 154 * work with hbase-2.x; it can read state but is not allowed to change state; i.e. effect 'repair'. 155 * Even though it can 'read' state, given how so much has changed in how hbase1 and hbase2 operate, 156 * it will often misread. See hbck2 (HBASE-19121) for a hbck tool for hbase2. This class is 157 * deprecated. 158 * 159 * <p> 160 * Region consistency checks verify that hbase:meta, region deployment on region 161 * servers and the state of data in HDFS (.regioninfo files) all are in 162 * accordance. 163 * <p> 164 * Table integrity checks verify that all possible row keys resolve to exactly 165 * one region of a table. This means there are no individual degenerate 166 * or backwards regions; no holes between regions; and that there are no 167 * overlapping regions. 168 * <p> 169 * The general repair strategy works in two phases: 170 * <ol> 171 * <li> Repair Table Integrity on HDFS. (merge or fabricate regions) 172 * <li> Repair Region Consistency with hbase:meta and assignments 173 * </ol> 174 * <p> 175 * For table integrity repairs, the tables' region directories are scanned 176 * for .regioninfo files. Each table's integrity is then verified. If there 177 * are any orphan regions (regions with no .regioninfo files) or holes, new 178 * regions are fabricated. Backwards regions are sidelined as well as empty 179 * degenerate (endkey==startkey) regions. If there are any overlapping regions, 180 * a new region is created and all data is merged into the new region. 181 * <p> 182 * Table integrity repairs deal solely with HDFS and could potentially be done 183 * offline -- the hbase region servers or master do not need to be running. 184 * This phase can eventually be used to completely reconstruct the hbase:meta table in 185 * an offline fashion. 186 * <p> 187 * Region consistency requires three conditions -- 1) valid .regioninfo file 188 * present in an HDFS region dir, 2) valid row with .regioninfo data in META, 189 * and 3) a region is deployed only at the regionserver that was assigned to 190 * with proper state in the master. 191 * <p> 192 * Region consistency repairs require hbase to be online so that hbck can 193 * contact the HBase master and region servers. The hbck#connect() method must 194 * first be called successfully. Much of the region consistency information 195 * is transient and less risky to repair. 196 * <p> 197 * If hbck is run from the command line, there are a handful of arguments that 198 * can be used to limit the kinds of repairs hbck will do. See the code in 199 * {@link #printUsageAndExit()} for more details. 200 * @deprecated For removal in hbase-4.0.0. Use HBCK2 instead. 201 */ 202@Deprecated 203@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.TOOLS) 204@InterfaceStability.Evolving 205public class HBaseFsck extends Configured implements Closeable { 206 public static final long DEFAULT_TIME_LAG = 60000; // default value of 1 minute 207 public static final long DEFAULT_SLEEP_BEFORE_RERUN = 10000; 208 private static final int MAX_NUM_THREADS = 50; // #threads to contact regions 209 private static boolean rsSupportsOffline = true; 210 private static final int DEFAULT_OVERLAPS_TO_SIDELINE = 2; 211 private static final int DEFAULT_MAX_MERGE = 5; 212 213 /** 214 * Here is where hbase-1.x used to default the lock for hbck1. 215 * It puts in place a lock when it goes to write/make changes. 216 */ 217 @VisibleForTesting 218 public static final String HBCK_LOCK_FILE = "hbase-hbck.lock"; 219 private static final int DEFAULT_MAX_LOCK_FILE_ATTEMPTS = 5; 220 private static final int DEFAULT_LOCK_FILE_ATTEMPT_SLEEP_INTERVAL = 200; // milliseconds 221 private static final int DEFAULT_LOCK_FILE_ATTEMPT_MAX_SLEEP_TIME = 5000; // milliseconds 222 // We have to set the timeout value > HdfsConstants.LEASE_SOFTLIMIT_PERIOD. 223 // In HADOOP-2.6 and later, the Namenode proxy now created with custom RetryPolicy for 224 // AlreadyBeingCreatedException which is implies timeout on this operations up to 225 // HdfsConstants.LEASE_SOFTLIMIT_PERIOD (60 seconds). 226 private static final int DEFAULT_WAIT_FOR_LOCK_TIMEOUT = 80; // seconds 227 private static final int DEFAULT_MAX_CREATE_ZNODE_ATTEMPTS = 5; 228 private static final int DEFAULT_CREATE_ZNODE_ATTEMPT_SLEEP_INTERVAL = 200; // milliseconds 229 private static final int DEFAULT_CREATE_ZNODE_ATTEMPT_MAX_SLEEP_TIME = 5000; // milliseconds 230 231 /********************** 232 * Internal resources 233 **********************/ 234 private static final Logger LOG = LoggerFactory.getLogger(HBaseFsck.class.getName()); 235 private ClusterMetrics status; 236 private ClusterConnection connection; 237 private Admin admin; 238 private Table meta; 239 // threads to do ||izable tasks: retrieve data from regionservers, handle overlapping regions 240 protected ExecutorService executor; 241 private long startMillis = EnvironmentEdgeManager.currentTime(); 242 private HFileCorruptionChecker hfcc; 243 private int retcode = 0; 244 private Path HBCK_LOCK_PATH; 245 private FSDataOutputStream hbckOutFd; 246 // This lock is to prevent cleanup of balancer resources twice between 247 // ShutdownHook and the main code. We cleanup only if the connect() is 248 // successful 249 private final AtomicBoolean hbckLockCleanup = new AtomicBoolean(false); 250 251 // Unsupported options in HBase 2.0+ 252 private static final Set<String> unsupportedOptionsInV2 = Sets.newHashSet("-fix", 253 "-fixAssignments", "-fixMeta", "-fixHdfsHoles", "-fixHdfsOrphans", "-fixTableOrphans", 254 "-fixHdfsOverlaps", "-sidelineBigOverlaps", "-fixSplitParents", "-removeParents", 255 "-fixEmptyMetaCells", "-repair", "-repairHoles", "-maxOverlapsToSideline", "-maxMerge"); 256 257 /*********** 258 * Options 259 ***********/ 260 private static boolean details = false; // do we display the full report 261 private long timelag = DEFAULT_TIME_LAG; // tables whose modtime is older 262 private static boolean forceExclusive = false; // only this hbck can modify HBase 263 private boolean fixAssignments = false; // fix assignment errors? 264 private boolean fixMeta = false; // fix meta errors? 265 private boolean checkHdfs = true; // load and check fs consistency? 266 private boolean fixHdfsHoles = false; // fix fs holes? 267 private boolean fixHdfsOverlaps = false; // fix fs overlaps (risky) 268 private boolean fixHdfsOrphans = false; // fix fs holes (missing .regioninfo) 269 private boolean fixTableOrphans = false; // fix fs holes (missing .tableinfo) 270 private boolean fixVersionFile = false; // fix missing hbase.version file in hdfs 271 private boolean fixSplitParents = false; // fix lingering split parents 272 private boolean removeParents = false; // remove split parents 273 private boolean fixReferenceFiles = false; // fix lingering reference store file 274 private boolean fixHFileLinks = false; // fix lingering HFileLinks 275 private boolean fixEmptyMetaCells = false; // fix (remove) empty REGIONINFO_QUALIFIER rows 276 private boolean fixReplication = false; // fix undeleted replication queues for removed peer 277 private boolean cleanReplicationBarrier = false; // clean replication barriers of a table 278 private boolean fixAny = false; // Set to true if any of the fix is required. 279 280 // limit checking/fixes to listed tables, if empty attempt to check/fix all 281 // hbase:meta are always checked 282 private Set<TableName> tablesIncluded = new HashSet<>(); 283 private TableName cleanReplicationBarrierTable; 284 private int maxMerge = DEFAULT_MAX_MERGE; // maximum number of overlapping regions to merge 285 // maximum number of overlapping regions to sideline 286 private int maxOverlapsToSideline = DEFAULT_OVERLAPS_TO_SIDELINE; 287 private boolean sidelineBigOverlaps = false; // sideline overlaps with >maxMerge regions 288 private Path sidelineDir = null; 289 290 private boolean rerun = false; // if we tried to fix something, rerun hbck 291 private static boolean summary = false; // if we want to print less output 292 private boolean checkMetaOnly = false; 293 private boolean checkRegionBoundaries = false; 294 private boolean ignorePreCheckPermission = false; // if pre-check permission 295 296 /********* 297 * State 298 *********/ 299 final private HbckErrorReporter errors; 300 int fixes = 0; 301 302 /** 303 * This map contains the state of all hbck items. It maps from encoded region 304 * name to HbckRegionInfo structure. The information contained in HbckRegionInfo is used 305 * to detect and correct consistency (hdfs/meta/deployment) problems. 306 */ 307 private TreeMap<String, HbckRegionInfo> regionInfoMap = new TreeMap<>(); 308 // Empty regioninfo qualifiers in hbase:meta 309 private Set<Result> emptyRegionInfoQualifiers = new HashSet<>(); 310 311 /** 312 * This map from Tablename -> TableInfo contains the structures necessary to 313 * detect table consistency problems (holes, dupes, overlaps). It is sorted 314 * to prevent dupes. 315 * 316 * If tablesIncluded is empty, this map contains all tables. 317 * Otherwise, it contains only meta tables and tables in tablesIncluded, 318 * unless checkMetaOnly is specified, in which case, it contains only 319 * the meta table 320 */ 321 private SortedMap<TableName, HbckTableInfo> tablesInfo = new ConcurrentSkipListMap<>(); 322 323 /** 324 * When initially looking at HDFS, we attempt to find any orphaned data. 325 */ 326 private List<HbckRegionInfo> orphanHdfsDirs = Collections.synchronizedList(new ArrayList<>()); 327 328 private Map<TableName, Set<String>> orphanTableDirs = new HashMap<>(); 329 private Map<TableName, TableState> tableStates = new HashMap<>(); 330 private final RetryCounterFactory lockFileRetryCounterFactory; 331 private final RetryCounterFactory createZNodeRetryCounterFactory; 332 333 private Map<TableName, Set<String>> skippedRegions = new HashMap<>(); 334 335 private ZKWatcher zkw = null; 336 private String hbckEphemeralNodePath = null; 337 private boolean hbckZodeCreated = false; 338 339 /** 340 * Constructor 341 * 342 * @param conf Configuration object 343 * @throws MasterNotRunningException if the master is not running 344 * @throws ZooKeeperConnectionException if unable to connect to ZooKeeper 345 */ 346 public HBaseFsck(Configuration conf) throws IOException, ClassNotFoundException { 347 this(conf, createThreadPool(conf)); 348 } 349 350 private static ExecutorService createThreadPool(Configuration conf) { 351 int numThreads = conf.getInt("hbasefsck.numthreads", MAX_NUM_THREADS); 352 return new ScheduledThreadPoolExecutor(numThreads, Threads.newDaemonThreadFactory("hbasefsck")); 353 } 354 355 /** 356 * Constructor 357 * 358 * @param conf 359 * Configuration object 360 * @throws MasterNotRunningException 361 * if the master is not running 362 * @throws ZooKeeperConnectionException 363 * if unable to connect to ZooKeeper 364 */ 365 public HBaseFsck(Configuration conf, ExecutorService exec) throws MasterNotRunningException, 366 ZooKeeperConnectionException, IOException, ClassNotFoundException { 367 super(conf); 368 errors = getErrorReporter(getConf()); 369 this.executor = exec; 370 lockFileRetryCounterFactory = createLockRetryCounterFactory(getConf()); 371 createZNodeRetryCounterFactory = createZnodeRetryCounterFactory(getConf()); 372 zkw = createZooKeeperWatcher(); 373 } 374 375 /** 376 * @return A retry counter factory configured for retrying lock file creation. 377 */ 378 public static RetryCounterFactory createLockRetryCounterFactory(Configuration conf) { 379 return new RetryCounterFactory( 380 conf.getInt("hbase.hbck.lockfile.attempts", DEFAULT_MAX_LOCK_FILE_ATTEMPTS), 381 conf.getInt("hbase.hbck.lockfile.attempt.sleep.interval", 382 DEFAULT_LOCK_FILE_ATTEMPT_SLEEP_INTERVAL), 383 conf.getInt("hbase.hbck.lockfile.attempt.maxsleeptime", 384 DEFAULT_LOCK_FILE_ATTEMPT_MAX_SLEEP_TIME)); 385 } 386 387 /** 388 * @return A retry counter factory configured for retrying znode creation. 389 */ 390 private static RetryCounterFactory createZnodeRetryCounterFactory(Configuration conf) { 391 return new RetryCounterFactory( 392 conf.getInt("hbase.hbck.createznode.attempts", DEFAULT_MAX_CREATE_ZNODE_ATTEMPTS), 393 conf.getInt("hbase.hbck.createznode.attempt.sleep.interval", 394 DEFAULT_CREATE_ZNODE_ATTEMPT_SLEEP_INTERVAL), 395 conf.getInt("hbase.hbck.createznode.attempt.maxsleeptime", 396 DEFAULT_CREATE_ZNODE_ATTEMPT_MAX_SLEEP_TIME)); 397 } 398 399 /** 400 * @return Return the tmp dir this tool writes too. 401 */ 402 @VisibleForTesting 403 public static Path getTmpDir(Configuration conf) throws IOException { 404 return new Path(CommonFSUtils.getRootDir(conf), HConstants.HBASE_TEMP_DIRECTORY); 405 } 406 407 private static class FileLockCallable implements Callable<FSDataOutputStream> { 408 RetryCounter retryCounter; 409 private final Configuration conf; 410 private Path hbckLockPath = null; 411 412 public FileLockCallable(Configuration conf, RetryCounter retryCounter) { 413 this.retryCounter = retryCounter; 414 this.conf = conf; 415 } 416 417 /** 418 * @return Will be <code>null</code> unless you call {@link #call()} 419 */ 420 Path getHbckLockPath() { 421 return this.hbckLockPath; 422 } 423 424 @Override 425 public FSDataOutputStream call() throws IOException { 426 try { 427 FileSystem fs = CommonFSUtils.getCurrentFileSystem(this.conf); 428 FsPermission defaultPerms = 429 CommonFSUtils.getFilePermissions(fs, this.conf, HConstants.DATA_FILE_UMASK_KEY); 430 Path tmpDir = getTmpDir(conf); 431 this.hbckLockPath = new Path(tmpDir, HBCK_LOCK_FILE); 432 fs.mkdirs(tmpDir); 433 final FSDataOutputStream out = createFileWithRetries(fs, this.hbckLockPath, defaultPerms); 434 out.writeBytes(InetAddress.getLocalHost().toString()); 435 // Add a note into the file we write on why hbase2 is writing out an hbck1 lock file. 436 out.writeBytes(" Written by an hbase-2.x Master to block an " + 437 "attempt by an hbase-1.x HBCK tool making modification to state. " + 438 "See 'HBCK must match HBase server version' in the hbase refguide."); 439 out.flush(); 440 return out; 441 } catch(RemoteException e) { 442 if(AlreadyBeingCreatedException.class.getName().equals(e.getClassName())){ 443 return null; 444 } else { 445 throw e; 446 } 447 } 448 } 449 450 private FSDataOutputStream createFileWithRetries(final FileSystem fs, 451 final Path hbckLockFilePath, final FsPermission defaultPerms) 452 throws IOException { 453 IOException exception = null; 454 do { 455 try { 456 return CommonFSUtils.create(fs, hbckLockFilePath, defaultPerms, false); 457 } catch (IOException ioe) { 458 LOG.info("Failed to create lock file " + hbckLockFilePath.getName() 459 + ", try=" + (retryCounter.getAttemptTimes() + 1) + " of " 460 + retryCounter.getMaxAttempts()); 461 LOG.debug("Failed to create lock file " + hbckLockFilePath.getName(), 462 ioe); 463 try { 464 exception = ioe; 465 retryCounter.sleepUntilNextRetry(); 466 } catch (InterruptedException ie) { 467 throw (InterruptedIOException) new InterruptedIOException( 468 "Can't create lock file " + hbckLockFilePath.getName()) 469 .initCause(ie); 470 } 471 } 472 } while (retryCounter.shouldRetry()); 473 474 throw exception; 475 } 476 } 477 478 /** 479 * This method maintains a lock using a file. If the creation fails we return null 480 * 481 * @return FSDataOutputStream object corresponding to the newly opened lock file 482 * @throws IOException if IO failure occurs 483 */ 484 public static Pair<Path, FSDataOutputStream> checkAndMarkRunningHbck(Configuration conf, 485 RetryCounter retryCounter) throws IOException { 486 FileLockCallable callable = new FileLockCallable(conf, retryCounter); 487 ExecutorService executor = Executors.newFixedThreadPool(1); 488 FutureTask<FSDataOutputStream> futureTask = new FutureTask<>(callable); 489 executor.execute(futureTask); 490 final int timeoutInSeconds = conf.getInt( 491 "hbase.hbck.lockfile.maxwaittime", DEFAULT_WAIT_FOR_LOCK_TIMEOUT); 492 FSDataOutputStream stream = null; 493 try { 494 stream = futureTask.get(timeoutInSeconds, TimeUnit.SECONDS); 495 } catch (ExecutionException ee) { 496 LOG.warn("Encountered exception when opening lock file", ee); 497 } catch (InterruptedException ie) { 498 LOG.warn("Interrupted when opening lock file", ie); 499 Thread.currentThread().interrupt(); 500 } catch (TimeoutException exception) { 501 // took too long to obtain lock 502 LOG.warn("Took more than " + timeoutInSeconds + " seconds in obtaining lock"); 503 futureTask.cancel(true); 504 } finally { 505 executor.shutdownNow(); 506 } 507 return new Pair<Path, FSDataOutputStream>(callable.getHbckLockPath(), stream); 508 } 509 510 private void unlockHbck() { 511 if (isExclusive() && hbckLockCleanup.compareAndSet(true, false)) { 512 RetryCounter retryCounter = lockFileRetryCounterFactory.create(); 513 do { 514 try { 515 IOUtils.closeQuietly(hbckOutFd); 516 CommonFSUtils.delete(CommonFSUtils.getCurrentFileSystem(getConf()), HBCK_LOCK_PATH, true); 517 LOG.info("Finishing hbck"); 518 return; 519 } catch (IOException ioe) { 520 LOG.info("Failed to delete " + HBCK_LOCK_PATH + ", try=" 521 + (retryCounter.getAttemptTimes() + 1) + " of " 522 + retryCounter.getMaxAttempts()); 523 LOG.debug("Failed to delete " + HBCK_LOCK_PATH, ioe); 524 try { 525 retryCounter.sleepUntilNextRetry(); 526 } catch (InterruptedException ie) { 527 Thread.currentThread().interrupt(); 528 LOG.warn("Interrupted while deleting lock file" + 529 HBCK_LOCK_PATH); 530 return; 531 } 532 } 533 } while (retryCounter.shouldRetry()); 534 } 535 } 536 537 /** 538 * To repair region consistency, one must call connect() in order to repair 539 * online state. 540 */ 541 public void connect() throws IOException { 542 543 if (isExclusive()) { 544 // Grab the lock 545 Pair<Path, FSDataOutputStream> pair = 546 checkAndMarkRunningHbck(getConf(), this.lockFileRetryCounterFactory.create()); 547 HBCK_LOCK_PATH = pair.getFirst(); 548 this.hbckOutFd = pair.getSecond(); 549 if (hbckOutFd == null) { 550 setRetCode(-1); 551 LOG.error("Another instance of hbck is fixing HBase, exiting this instance. " + 552 "[If you are sure no other instance is running, delete the lock file " + 553 HBCK_LOCK_PATH + " and rerun the tool]"); 554 throw new IOException("Duplicate hbck - Abort"); 555 } 556 557 // Make sure to cleanup the lock 558 hbckLockCleanup.set(true); 559 } 560 561 562 // Add a shutdown hook to this thread, in case user tries to 563 // kill the hbck with a ctrl-c, we want to cleanup the lock so that 564 // it is available for further calls 565 Runtime.getRuntime().addShutdownHook(new Thread() { 566 @Override 567 public void run() { 568 IOUtils.closeQuietly(HBaseFsck.this); 569 cleanupHbckZnode(); 570 unlockHbck(); 571 } 572 }); 573 574 LOG.info("Launching hbck"); 575 576 connection = (ClusterConnection)ConnectionFactory.createConnection(getConf()); 577 admin = connection.getAdmin(); 578 meta = connection.getTable(TableName.META_TABLE_NAME); 579 status = admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS, 580 Option.DEAD_SERVERS, Option.MASTER, Option.BACKUP_MASTERS, 581 Option.REGIONS_IN_TRANSITION, Option.HBASE_VERSION)); 582 } 583 584 /** 585 * Get deployed regions according to the region servers. 586 */ 587 private void loadDeployedRegions() throws IOException, InterruptedException { 588 // From the master, get a list of all known live region servers 589 Collection<ServerName> regionServers = status.getLiveServerMetrics().keySet(); 590 errors.print("Number of live region servers: " + regionServers.size()); 591 if (details) { 592 for (ServerName rsinfo: regionServers) { 593 errors.print(" " + rsinfo.getServerName()); 594 } 595 } 596 597 // From the master, get a list of all dead region servers 598 Collection<ServerName> deadRegionServers = status.getDeadServerNames(); 599 errors.print("Number of dead region servers: " + deadRegionServers.size()); 600 if (details) { 601 for (ServerName name: deadRegionServers) { 602 errors.print(" " + name); 603 } 604 } 605 606 // Print the current master name and state 607 errors.print("Master: " + status.getMasterName()); 608 609 // Print the list of all backup masters 610 Collection<ServerName> backupMasters = status.getBackupMasterNames(); 611 errors.print("Number of backup masters: " + backupMasters.size()); 612 if (details) { 613 for (ServerName name: backupMasters) { 614 errors.print(" " + name); 615 } 616 } 617 618 errors.print("Average load: " + status.getAverageLoad()); 619 errors.print("Number of requests: " + status.getRequestCount()); 620 errors.print("Number of regions: " + status.getRegionCount()); 621 622 List<RegionState> rits = status.getRegionStatesInTransition(); 623 errors.print("Number of regions in transition: " + rits.size()); 624 if (details) { 625 for (RegionState state: rits) { 626 errors.print(" " + state.toDescriptiveString()); 627 } 628 } 629 630 // Determine what's deployed 631 processRegionServers(regionServers); 632 } 633 634 /** 635 * Clear the current state of hbck. 636 */ 637 private void clearState() { 638 // Make sure regionInfo is empty before starting 639 fixes = 0; 640 regionInfoMap.clear(); 641 emptyRegionInfoQualifiers.clear(); 642 tableStates.clear(); 643 errors.clear(); 644 tablesInfo.clear(); 645 orphanHdfsDirs.clear(); 646 skippedRegions.clear(); 647 } 648 649 /** 650 * This repair method analyzes hbase data in hdfs and repairs it to satisfy 651 * the table integrity rules. HBase doesn't need to be online for this 652 * operation to work. 653 */ 654 public void offlineHdfsIntegrityRepair() throws IOException, InterruptedException { 655 // Initial pass to fix orphans. 656 if (shouldCheckHdfs() && (shouldFixHdfsOrphans() || shouldFixHdfsHoles() 657 || shouldFixHdfsOverlaps() || shouldFixTableOrphans())) { 658 LOG.info("Loading regioninfos HDFS"); 659 // if nothing is happening this should always complete in two iterations. 660 int maxIterations = getConf().getInt("hbase.hbck.integrityrepair.iterations.max", 3); 661 int curIter = 0; 662 do { 663 clearState(); // clears hbck state and reset fixes to 0 and. 664 // repair what's on HDFS 665 restoreHdfsIntegrity(); 666 curIter++;// limit the number of iterations. 667 } while (fixes > 0 && curIter <= maxIterations); 668 669 // Repairs should be done in the first iteration and verification in the second. 670 // If there are more than 2 passes, something funny has happened. 671 if (curIter > 2) { 672 if (curIter == maxIterations) { 673 LOG.warn("Exiting integrity repairs after max " + curIter + " iterations. " 674 + "Tables integrity may not be fully repaired!"); 675 } else { 676 LOG.info("Successfully exiting integrity repairs after " + curIter + " iterations"); 677 } 678 } 679 } 680 } 681 682 /** 683 * This repair method requires the cluster to be online since it contacts 684 * region servers and the masters. It makes each region's state in HDFS, in 685 * hbase:meta, and deployments consistent. 686 * 687 * @return If > 0 , number of errors detected, if < 0 there was an unrecoverable 688 * error. If 0, we have a clean hbase. 689 */ 690 public int onlineConsistencyRepair() throws IOException, KeeperException, 691 InterruptedException { 692 693 // get regions according to what is online on each RegionServer 694 loadDeployedRegions(); 695 // check whether hbase:meta is deployed and online 696 recordMetaRegion(); 697 // Check if hbase:meta is found only once and in the right place 698 if (!checkMetaRegion()) { 699 String errorMsg = "hbase:meta table is not consistent. "; 700 if (shouldFixAssignments()) { 701 errorMsg += "HBCK will try fixing it. Rerun once hbase:meta is back to consistent state."; 702 } else { 703 errorMsg += "Run HBCK with proper fix options to fix hbase:meta inconsistency."; 704 } 705 errors.reportError(errorMsg + " Exiting..."); 706 return -2; 707 } 708 // Not going with further consistency check for tables when hbase:meta itself is not consistent. 709 LOG.info("Loading regionsinfo from the hbase:meta table"); 710 boolean success = loadMetaEntries(); 711 if (!success) return -1; 712 713 // Empty cells in hbase:meta? 714 reportEmptyMetaCells(); 715 716 // Check if we have to cleanup empty REGIONINFO_QUALIFIER rows from hbase:meta 717 if (shouldFixEmptyMetaCells()) { 718 fixEmptyMetaCells(); 719 } 720 721 // get a list of all tables that have not changed recently. 722 if (!checkMetaOnly) { 723 reportTablesInFlux(); 724 } 725 726 // Get disabled tables states 727 loadTableStates(); 728 729 // load regiondirs and regioninfos from HDFS 730 if (shouldCheckHdfs()) { 731 LOG.info("Loading region directories from HDFS"); 732 loadHdfsRegionDirs(); 733 LOG.info("Loading region information from HDFS"); 734 loadHdfsRegionInfos(); 735 } 736 737 // fix the orphan tables 738 fixOrphanTables(); 739 740 LOG.info("Checking and fixing region consistency"); 741 // Check and fix consistency 742 checkAndFixConsistency(); 743 744 // Check integrity (does not fix) 745 checkIntegrity(); 746 return errors.getErrorList().size(); 747 } 748 749 /** 750 * This method maintains an ephemeral znode. If the creation fails we return false or throw 751 * exception 752 * 753 * @return true if creating znode succeeds; false otherwise 754 * @throws IOException if IO failure occurs 755 */ 756 private boolean setMasterInMaintenanceMode() throws IOException { 757 RetryCounter retryCounter = createZNodeRetryCounterFactory.create(); 758 hbckEphemeralNodePath = ZNodePaths.joinZNode( 759 zkw.getZNodePaths().masterMaintZNode, 760 "hbck-" + Long.toString(EnvironmentEdgeManager.currentTime())); 761 do { 762 try { 763 hbckZodeCreated = ZKUtil.createEphemeralNodeAndWatch(zkw, hbckEphemeralNodePath, null); 764 if (hbckZodeCreated) { 765 break; 766 } 767 } catch (KeeperException e) { 768 if (retryCounter.getAttemptTimes() >= retryCounter.getMaxAttempts()) { 769 throw new IOException("Can't create znode " + hbckEphemeralNodePath, e); 770 } 771 // fall through and retry 772 } 773 774 LOG.warn("Fail to create znode " + hbckEphemeralNodePath + ", try=" + 775 (retryCounter.getAttemptTimes() + 1) + " of " + retryCounter.getMaxAttempts()); 776 777 try { 778 retryCounter.sleepUntilNextRetry(); 779 } catch (InterruptedException ie) { 780 throw (InterruptedIOException) new InterruptedIOException( 781 "Can't create znode " + hbckEphemeralNodePath).initCause(ie); 782 } 783 } while (retryCounter.shouldRetry()); 784 return hbckZodeCreated; 785 } 786 787 private void cleanupHbckZnode() { 788 try { 789 if (zkw != null && hbckZodeCreated) { 790 ZKUtil.deleteNode(zkw, hbckEphemeralNodePath); 791 hbckZodeCreated = false; 792 } 793 } catch (KeeperException e) { 794 // Ignore 795 if (!e.code().equals(KeeperException.Code.NONODE)) { 796 LOG.warn("Delete HBCK znode " + hbckEphemeralNodePath + " failed ", e); 797 } 798 } 799 } 800 801 /** 802 * Contacts the master and prints out cluster-wide information 803 * @return 0 on success, non-zero on failure 804 */ 805 public int onlineHbck() 806 throws IOException, KeeperException, InterruptedException, ReplicationException { 807 // print hbase server version 808 errors.print("Version: " + status.getHBaseVersion()); 809 810 // Clean start 811 clearState(); 812 // Do offline check and repair first 813 offlineHdfsIntegrityRepair(); 814 offlineReferenceFileRepair(); 815 offlineHLinkFileRepair(); 816 // If Master runs maintenance tasks (such as balancer, catalog janitor, etc) during online 817 // hbck, it is likely that hbck would be misled and report transient errors. Therefore, it 818 // is better to set Master into maintenance mode during online hbck. 819 // 820 if (!setMasterInMaintenanceMode()) { 821 LOG.warn("HBCK is running while master is not in maintenance mode, you might see transient " 822 + "error. Please run HBCK multiple times to reduce the chance of transient error."); 823 } 824 825 onlineConsistencyRepair(); 826 827 if (checkRegionBoundaries) { 828 checkRegionBoundaries(); 829 } 830 831 checkAndFixReplication(); 832 833 cleanReplicationBarrier(); 834 835 // Remove the hbck znode 836 cleanupHbckZnode(); 837 838 // Remove the hbck lock 839 unlockHbck(); 840 841 // Print table summary 842 printTableSummary(tablesInfo); 843 return errors.summarize(); 844 } 845 846 public static byte[] keyOnly(byte[] b) { 847 if (b == null) 848 return b; 849 int rowlength = Bytes.toShort(b, 0); 850 byte[] result = new byte[rowlength]; 851 System.arraycopy(b, Bytes.SIZEOF_SHORT, result, 0, rowlength); 852 return result; 853 } 854 855 @Override 856 public void close() throws IOException { 857 try { 858 cleanupHbckZnode(); 859 unlockHbck(); 860 } catch (Exception io) { 861 LOG.warn(io.toString(), io); 862 } finally { 863 if (zkw != null) { 864 zkw.close(); 865 zkw = null; 866 } 867 IOUtils.closeQuietly(admin); 868 IOUtils.closeQuietly(meta); 869 IOUtils.closeQuietly(connection); 870 } 871 } 872 873 private static class RegionBoundariesInformation { 874 public byte [] regionName; 875 public byte [] metaFirstKey; 876 public byte [] metaLastKey; 877 public byte [] storesFirstKey; 878 public byte [] storesLastKey; 879 @Override 880 public String toString () { 881 return "regionName=" + Bytes.toStringBinary(regionName) + 882 "\nmetaFirstKey=" + Bytes.toStringBinary(metaFirstKey) + 883 "\nmetaLastKey=" + Bytes.toStringBinary(metaLastKey) + 884 "\nstoresFirstKey=" + Bytes.toStringBinary(storesFirstKey) + 885 "\nstoresLastKey=" + Bytes.toStringBinary(storesLastKey); 886 } 887 } 888 889 public void checkRegionBoundaries() { 890 try { 891 ByteArrayComparator comparator = new ByteArrayComparator(); 892 List<RegionInfo> regions = MetaTableAccessor.getAllRegions(connection, true); 893 final RegionBoundariesInformation currentRegionBoundariesInformation = 894 new RegionBoundariesInformation(); 895 Path hbaseRoot = CommonFSUtils.getRootDir(getConf()); 896 for (RegionInfo regionInfo : regions) { 897 Path tableDir = CommonFSUtils.getTableDir(hbaseRoot, regionInfo.getTable()); 898 currentRegionBoundariesInformation.regionName = regionInfo.getRegionName(); 899 // For each region, get the start and stop key from the META and compare them to the 900 // same information from the Stores. 901 Path path = new Path(tableDir, regionInfo.getEncodedName()); 902 FileSystem fs = path.getFileSystem(getConf()); 903 FileStatus[] files = fs.listStatus(path); 904 // For all the column families in this region... 905 byte[] storeFirstKey = null; 906 byte[] storeLastKey = null; 907 for (FileStatus file : files) { 908 String fileName = file.getPath().toString(); 909 fileName = fileName.substring(fileName.lastIndexOf("/") + 1); 910 if (!fileName.startsWith(".") && !fileName.endsWith("recovered.edits")) { 911 FileStatus[] storeFiles = fs.listStatus(file.getPath()); 912 // For all the stores in this column family. 913 for (FileStatus storeFile : storeFiles) { 914 HFile.Reader reader = HFile.createReader(fs, storeFile.getPath(), 915 CacheConfig.DISABLED, true, getConf()); 916 if ((reader.getFirstKey() != null) 917 && ((storeFirstKey == null) || (comparator.compare(storeFirstKey, 918 ((KeyValue.KeyOnlyKeyValue) reader.getFirstKey().get()).getKey()) > 0))) { 919 storeFirstKey = ((KeyValue.KeyOnlyKeyValue)reader.getFirstKey().get()).getKey(); 920 } 921 if ((reader.getLastKey() != null) 922 && ((storeLastKey == null) || (comparator.compare(storeLastKey, 923 ((KeyValue.KeyOnlyKeyValue)reader.getLastKey().get()).getKey())) < 0)) { 924 storeLastKey = ((KeyValue.KeyOnlyKeyValue)reader.getLastKey().get()).getKey(); 925 } 926 reader.close(); 927 } 928 } 929 } 930 currentRegionBoundariesInformation.metaFirstKey = regionInfo.getStartKey(); 931 currentRegionBoundariesInformation.metaLastKey = regionInfo.getEndKey(); 932 currentRegionBoundariesInformation.storesFirstKey = keyOnly(storeFirstKey); 933 currentRegionBoundariesInformation.storesLastKey = keyOnly(storeLastKey); 934 if (currentRegionBoundariesInformation.metaFirstKey.length == 0) 935 currentRegionBoundariesInformation.metaFirstKey = null; 936 if (currentRegionBoundariesInformation.metaLastKey.length == 0) 937 currentRegionBoundariesInformation.metaLastKey = null; 938 939 // For a region to be correct, we need the META start key to be smaller or equal to the 940 // smallest start key from all the stores, and the start key from the next META entry to 941 // be bigger than the last key from all the current stores. First region start key is null; 942 // Last region end key is null; some regions can be empty and not have any store. 943 944 boolean valid = true; 945 // Checking start key. 946 if ((currentRegionBoundariesInformation.storesFirstKey != null) 947 && (currentRegionBoundariesInformation.metaFirstKey != null)) { 948 valid = valid 949 && comparator.compare(currentRegionBoundariesInformation.storesFirstKey, 950 currentRegionBoundariesInformation.metaFirstKey) >= 0; 951 } 952 // Checking stop key. 953 if ((currentRegionBoundariesInformation.storesLastKey != null) 954 && (currentRegionBoundariesInformation.metaLastKey != null)) { 955 valid = valid 956 && comparator.compare(currentRegionBoundariesInformation.storesLastKey, 957 currentRegionBoundariesInformation.metaLastKey) < 0; 958 } 959 if (!valid) { 960 errors.reportError(ERROR_CODE.BOUNDARIES_ERROR, "Found issues with regions boundaries", 961 tablesInfo.get(regionInfo.getTable())); 962 LOG.warn("Region's boundaries not aligned between stores and META for:"); 963 LOG.warn(Objects.toString(currentRegionBoundariesInformation)); 964 } 965 } 966 } catch (IOException e) { 967 LOG.error(e.toString(), e); 968 } 969 } 970 971 /** 972 * Iterates through the list of all orphan/invalid regiondirs. 973 */ 974 private void adoptHdfsOrphans(Collection<HbckRegionInfo> orphanHdfsDirs) throws IOException { 975 for (HbckRegionInfo hi : orphanHdfsDirs) { 976 LOG.info("Attempting to handle orphan hdfs dir: " + hi.getHdfsRegionDir()); 977 adoptHdfsOrphan(hi); 978 } 979 } 980 981 /** 982 * Orphaned regions are regions without a .regioninfo file in them. We "adopt" 983 * these orphans by creating a new region, and moving the column families, 984 * recovered edits, WALs, into the new region dir. We determine the region 985 * startkey and endkeys by looking at all of the hfiles inside the column 986 * families to identify the min and max keys. The resulting region will 987 * likely violate table integrity but will be dealt with by merging 988 * overlapping regions. 989 */ 990 @SuppressWarnings("deprecation") 991 private void adoptHdfsOrphan(HbckRegionInfo hi) throws IOException { 992 Path p = hi.getHdfsRegionDir(); 993 FileSystem fs = p.getFileSystem(getConf()); 994 FileStatus[] dirs = fs.listStatus(p); 995 if (dirs == null) { 996 LOG.warn("Attempt to adopt orphan hdfs region skipped because no files present in " + 997 p + ". This dir could probably be deleted."); 998 return ; 999 } 1000 1001 TableName tableName = hi.getTableName(); 1002 HbckTableInfo tableInfo = tablesInfo.get(tableName); 1003 Preconditions.checkNotNull(tableInfo, "Table '" + tableName + "' not present!"); 1004 TableDescriptor template = tableInfo.getTableDescriptor(); 1005 1006 // find min and max key values 1007 Pair<byte[],byte[]> orphanRegionRange = null; 1008 for (FileStatus cf : dirs) { 1009 String cfName= cf.getPath().getName(); 1010 // TODO Figure out what the special dirs are 1011 if (cfName.startsWith(".") || cfName.equals(HConstants.SPLIT_LOGDIR_NAME)) continue; 1012 1013 FileStatus[] hfiles = fs.listStatus(cf.getPath()); 1014 for (FileStatus hfile : hfiles) { 1015 byte[] start, end; 1016 HFile.Reader hf = null; 1017 try { 1018 hf = HFile.createReader(fs, hfile.getPath(), CacheConfig.DISABLED, true, getConf()); 1019 Optional<Cell> startKv = hf.getFirstKey(); 1020 start = CellUtil.cloneRow(startKv.get()); 1021 Optional<Cell> endKv = hf.getLastKey(); 1022 end = CellUtil.cloneRow(endKv.get()); 1023 } catch (IOException ioe) { 1024 LOG.warn("Problem reading orphan file " + hfile + ", skipping"); 1025 continue; 1026 } catch (NullPointerException ioe) { 1027 LOG.warn("Orphan file " + hfile + " is possibly corrupted HFile, skipping"); 1028 continue; 1029 } finally { 1030 if (hf != null) { 1031 hf.close(); 1032 } 1033 } 1034 1035 // expand the range to include the range of all hfiles 1036 if (orphanRegionRange == null) { 1037 // first range 1038 orphanRegionRange = new Pair<>(start, end); 1039 } else { 1040 // TODO add test 1041 1042 // expand range only if the hfile is wider. 1043 if (Bytes.compareTo(orphanRegionRange.getFirst(), start) > 0) { 1044 orphanRegionRange.setFirst(start); 1045 } 1046 if (Bytes.compareTo(orphanRegionRange.getSecond(), end) < 0 ) { 1047 orphanRegionRange.setSecond(end); 1048 } 1049 } 1050 } 1051 } 1052 if (orphanRegionRange == null) { 1053 LOG.warn("No data in dir " + p + ", sidelining data"); 1054 fixes++; 1055 sidelineRegionDir(fs, hi); 1056 return; 1057 } 1058 LOG.info("Min max keys are : [" + Bytes.toString(orphanRegionRange.getFirst()) + ", " + 1059 Bytes.toString(orphanRegionRange.getSecond()) + ")"); 1060 1061 // create new region on hdfs. move data into place. 1062 RegionInfo regionInfo = RegionInfoBuilder.newBuilder(template.getTableName()) 1063 .setStartKey(orphanRegionRange.getFirst()) 1064 .setEndKey(Bytes.add(orphanRegionRange.getSecond(), new byte[1])) 1065 .build(); 1066 LOG.info("Creating new region : " + regionInfo); 1067 HRegion region = HBaseFsckRepair.createHDFSRegionDir(getConf(), regionInfo, template); 1068 Path target = region.getRegionFileSystem().getRegionDir(); 1069 1070 // rename all the data to new region 1071 mergeRegionDirs(target, hi); 1072 fixes++; 1073 } 1074 1075 /** 1076 * This method determines if there are table integrity errors in HDFS. If 1077 * there are errors and the appropriate "fix" options are enabled, the method 1078 * will first correct orphan regions making them into legit regiondirs, and 1079 * then reload to merge potentially overlapping regions. 1080 * 1081 * @return number of table integrity errors found 1082 */ 1083 private int restoreHdfsIntegrity() throws IOException, InterruptedException { 1084 // Determine what's on HDFS 1085 LOG.info("Loading HBase regioninfo from HDFS..."); 1086 loadHdfsRegionDirs(); // populating regioninfo table. 1087 1088 int errs = errors.getErrorList().size(); 1089 // First time just get suggestions. 1090 tablesInfo = loadHdfsRegionInfos(); // update tableInfos based on region info in fs. 1091 checkHdfsIntegrity(false, false); 1092 1093 if (errors.getErrorList().size() == errs) { 1094 LOG.info("No integrity errors. We are done with this phase. Glorious."); 1095 return 0; 1096 } 1097 1098 if (shouldFixHdfsOrphans() && orphanHdfsDirs.size() > 0) { 1099 adoptHdfsOrphans(orphanHdfsDirs); 1100 // TODO optimize by incrementally adding instead of reloading. 1101 } 1102 1103 // Make sure there are no holes now. 1104 if (shouldFixHdfsHoles()) { 1105 clearState(); // this also resets # fixes. 1106 loadHdfsRegionDirs(); 1107 tablesInfo = loadHdfsRegionInfos(); // update tableInfos based on region info in fs. 1108 tablesInfo = checkHdfsIntegrity(shouldFixHdfsHoles(), false); 1109 } 1110 1111 // Now we fix overlaps 1112 if (shouldFixHdfsOverlaps()) { 1113 // second pass we fix overlaps. 1114 clearState(); // this also resets # fixes. 1115 loadHdfsRegionDirs(); 1116 tablesInfo = loadHdfsRegionInfos(); // update tableInfos based on region info in fs. 1117 tablesInfo = checkHdfsIntegrity(false, shouldFixHdfsOverlaps()); 1118 } 1119 1120 return errors.getErrorList().size(); 1121 } 1122 1123 /** 1124 * Scan all the store file names to find any lingering reference files, 1125 * which refer to some none-exiting files. If "fix" option is enabled, 1126 * any lingering reference file will be sidelined if found. 1127 * <p> 1128 * Lingering reference file prevents a region from opening. It has to 1129 * be fixed before a cluster can start properly. 1130 */ 1131 private void offlineReferenceFileRepair() throws IOException, InterruptedException { 1132 clearState(); 1133 Configuration conf = getConf(); 1134 Path hbaseRoot = CommonFSUtils.getRootDir(conf); 1135 FileSystem fs = hbaseRoot.getFileSystem(conf); 1136 LOG.info("Computing mapping of all store files"); 1137 Map<String, Path> allFiles = FSUtils.getTableStoreFilePathMap(fs, hbaseRoot, 1138 new FSUtils.ReferenceFileFilter(fs), executor, errors); 1139 errors.print(""); 1140 LOG.info("Validating mapping using HDFS state"); 1141 for (Path path: allFiles.values()) { 1142 Path referredToFile = StoreFileInfo.getReferredToFile(path); 1143 if (fs.exists(referredToFile)) continue; // good, expected 1144 1145 // Found a lingering reference file 1146 errors.reportError(ERROR_CODE.LINGERING_REFERENCE_HFILE, 1147 "Found lingering reference file " + path); 1148 if (!shouldFixReferenceFiles()) continue; 1149 1150 // Now, trying to fix it since requested 1151 boolean success = false; 1152 String pathStr = path.toString(); 1153 1154 // A reference file path should be like 1155 // ${hbase.rootdir}/data/namespace/table_name/region_id/family_name/referred_file.region_name 1156 // Up 5 directories to get the root folder. 1157 // So the file will be sidelined to a similar folder structure. 1158 int index = pathStr.lastIndexOf(Path.SEPARATOR_CHAR); 1159 for (int i = 0; index > 0 && i < 5; i++) { 1160 index = pathStr.lastIndexOf(Path.SEPARATOR_CHAR, index - 1); 1161 } 1162 if (index > 0) { 1163 Path rootDir = getSidelineDir(); 1164 Path dst = new Path(rootDir, pathStr.substring(index + 1)); 1165 fs.mkdirs(dst.getParent()); 1166 LOG.info("Trying to sideline reference file " 1167 + path + " to " + dst); 1168 setShouldRerun(); 1169 1170 success = fs.rename(path, dst); 1171 debugLsr(dst); 1172 1173 } 1174 if (!success) { 1175 LOG.error("Failed to sideline reference file " + path); 1176 } 1177 } 1178 } 1179 1180 /** 1181 * Scan all the store file names to find any lingering HFileLink files, 1182 * which refer to some none-exiting files. If "fix" option is enabled, 1183 * any lingering HFileLink file will be sidelined if found. 1184 */ 1185 private void offlineHLinkFileRepair() throws IOException, InterruptedException { 1186 Configuration conf = getConf(); 1187 Path hbaseRoot = CommonFSUtils.getRootDir(conf); 1188 FileSystem fs = hbaseRoot.getFileSystem(conf); 1189 LOG.info("Computing mapping of all link files"); 1190 Map<String, Path> allFiles = FSUtils 1191 .getTableStoreFilePathMap(fs, hbaseRoot, new FSUtils.HFileLinkFilter(), executor, errors); 1192 errors.print(""); 1193 1194 LOG.info("Validating mapping using HDFS state"); 1195 for (Path path : allFiles.values()) { 1196 // building HFileLink object to gather locations 1197 HFileLink actualLink = HFileLink.buildFromHFileLinkPattern(conf, path); 1198 if (actualLink.exists(fs)) continue; // good, expected 1199 1200 // Found a lingering HFileLink 1201 errors.reportError(ERROR_CODE.LINGERING_HFILELINK, "Found lingering HFileLink " + path); 1202 if (!shouldFixHFileLinks()) continue; 1203 1204 // Now, trying to fix it since requested 1205 setShouldRerun(); 1206 1207 // An HFileLink path should be like 1208 // ${hbase.rootdir}/data/namespace/table_name/region_id/family_name/linkedtable=linkedregionname-linkedhfilename 1209 // sidelineing will happen in the ${hbase.rootdir}/${sidelinedir} directory with the same folder structure. 1210 boolean success = sidelineFile(fs, hbaseRoot, path); 1211 1212 if (!success) { 1213 LOG.error("Failed to sideline HFileLink file " + path); 1214 } 1215 1216 // An HFileLink backreference path should be like 1217 // ${hbase.rootdir}/archive/data/namespace/table_name/region_id/family_name/.links-linkedhfilename 1218 // sidelineing will happen in the ${hbase.rootdir}/${sidelinedir} directory with the same folder structure. 1219 Path backRefPath = FileLink.getBackReferencesDir(HFileArchiveUtil 1220 .getStoreArchivePath(conf, HFileLink.getReferencedTableName(path.getName().toString()), 1221 HFileLink.getReferencedRegionName(path.getName().toString()), 1222 path.getParent().getName()), 1223 HFileLink.getReferencedHFileName(path.getName().toString())); 1224 success = sidelineFile(fs, hbaseRoot, backRefPath); 1225 1226 if (!success) { 1227 LOG.error("Failed to sideline HFileLink backreference file " + path); 1228 } 1229 } 1230 } 1231 1232 private boolean sidelineFile(FileSystem fs, Path hbaseRoot, Path path) throws IOException { 1233 URI uri = hbaseRoot.toUri().relativize(path.toUri()); 1234 if (uri.isAbsolute()) return false; 1235 String relativePath = uri.getPath(); 1236 Path rootDir = getSidelineDir(); 1237 Path dst = new Path(rootDir, relativePath); 1238 boolean pathCreated = fs.mkdirs(dst.getParent()); 1239 if (!pathCreated) { 1240 LOG.error("Failed to create path: " + dst.getParent()); 1241 return false; 1242 } 1243 LOG.info("Trying to sideline file " + path + " to " + dst); 1244 return fs.rename(path, dst); 1245 } 1246 1247 /** 1248 * TODO -- need to add tests for this. 1249 */ 1250 private void reportEmptyMetaCells() { 1251 errors.print("Number of empty REGIONINFO_QUALIFIER rows in hbase:meta: " + 1252 emptyRegionInfoQualifiers.size()); 1253 if (details) { 1254 for (Result r: emptyRegionInfoQualifiers) { 1255 errors.print(" " + r); 1256 } 1257 } 1258 } 1259 1260 /** 1261 * TODO -- need to add tests for this. 1262 */ 1263 private void reportTablesInFlux() { 1264 AtomicInteger numSkipped = new AtomicInteger(0); 1265 TableDescriptor[] allTables = getTables(numSkipped); 1266 errors.print("Number of Tables: " + allTables.length); 1267 if (details) { 1268 if (numSkipped.get() > 0) { 1269 errors.detail("Number of Tables in flux: " + numSkipped.get()); 1270 } 1271 for (TableDescriptor td : allTables) { 1272 errors.detail(" Table: " + td.getTableName() + "\t" + 1273 (td.isReadOnly() ? "ro" : "rw") + "\t" + 1274 (td.isMetaRegion() ? "META" : " ") + "\t" + 1275 " families: " + td.getColumnFamilyCount()); 1276 } 1277 } 1278 } 1279 1280 public HbckErrorReporter getErrors() { 1281 return errors; 1282 } 1283 1284 /** 1285 * Populate hbi's from regionInfos loaded from file system. 1286 */ 1287 private SortedMap<TableName, HbckTableInfo> loadHdfsRegionInfos() 1288 throws IOException, InterruptedException { 1289 tablesInfo.clear(); // regenerating the data 1290 // generate region split structure 1291 Collection<HbckRegionInfo> hbckRegionInfos = regionInfoMap.values(); 1292 1293 // Parallelized read of .regioninfo files. 1294 List<WorkItemHdfsRegionInfo> hbis = new ArrayList<>(hbckRegionInfos.size()); 1295 List<Future<Void>> hbiFutures; 1296 1297 for (HbckRegionInfo hbi : hbckRegionInfos) { 1298 WorkItemHdfsRegionInfo work = new WorkItemHdfsRegionInfo(hbi, this, errors); 1299 hbis.add(work); 1300 } 1301 1302 // Submit and wait for completion 1303 hbiFutures = executor.invokeAll(hbis); 1304 1305 for(int i=0; i<hbiFutures.size(); i++) { 1306 WorkItemHdfsRegionInfo work = hbis.get(i); 1307 Future<Void> f = hbiFutures.get(i); 1308 try { 1309 f.get(); 1310 } catch(ExecutionException e) { 1311 LOG.warn("Failed to read .regioninfo file for region " + 1312 work.hbi.getRegionNameAsString(), e.getCause()); 1313 } 1314 } 1315 1316 Path hbaseRoot = CommonFSUtils.getRootDir(getConf()); 1317 FileSystem fs = hbaseRoot.getFileSystem(getConf()); 1318 // serialized table info gathering. 1319 for (HbckRegionInfo hbi: hbckRegionInfos) { 1320 1321 if (hbi.getHdfsHRI() == null) { 1322 // was an orphan 1323 continue; 1324 } 1325 1326 1327 // get table name from hdfs, populate various HBaseFsck tables. 1328 TableName tableName = hbi.getTableName(); 1329 if (tableName == null) { 1330 // There was an entry in hbase:meta not in the HDFS? 1331 LOG.warn("tableName was null for: " + hbi); 1332 continue; 1333 } 1334 1335 HbckTableInfo modTInfo = tablesInfo.get(tableName); 1336 if (modTInfo == null) { 1337 // only executed once per table. 1338 modTInfo = new HbckTableInfo(tableName, this); 1339 tablesInfo.put(tableName, modTInfo); 1340 try { 1341 TableDescriptor htd = 1342 FSTableDescriptors.getTableDescriptorFromFs(fs, hbaseRoot, tableName); 1343 modTInfo.htds.add(htd); 1344 } catch (IOException ioe) { 1345 if (!orphanTableDirs.containsKey(tableName)) { 1346 LOG.warn("Unable to read .tableinfo from " + hbaseRoot, ioe); 1347 //should only report once for each table 1348 errors.reportError(ERROR_CODE.NO_TABLEINFO_FILE, 1349 "Unable to read .tableinfo from " + hbaseRoot + "/" + tableName); 1350 Set<String> columns = new HashSet<>(); 1351 orphanTableDirs.put(tableName, getColumnFamilyList(columns, hbi)); 1352 } 1353 } 1354 } 1355 if (!hbi.isSkipChecks()) { 1356 modTInfo.addRegionInfo(hbi); 1357 } 1358 } 1359 1360 loadTableInfosForTablesWithNoRegion(); 1361 errors.print(""); 1362 1363 return tablesInfo; 1364 } 1365 1366 /** 1367 * To get the column family list according to the column family dirs 1368 * @param columns 1369 * @param hbi 1370 * @return a set of column families 1371 * @throws IOException 1372 */ 1373 private Set<String> getColumnFamilyList(Set<String> columns, HbckRegionInfo hbi) 1374 throws IOException { 1375 Path regionDir = hbi.getHdfsRegionDir(); 1376 FileSystem fs = regionDir.getFileSystem(getConf()); 1377 FileStatus[] subDirs = fs.listStatus(regionDir, new FSUtils.FamilyDirFilter(fs)); 1378 for (FileStatus subdir : subDirs) { 1379 String columnfamily = subdir.getPath().getName(); 1380 columns.add(columnfamily); 1381 } 1382 return columns; 1383 } 1384 1385 /** 1386 * To fabricate a .tableinfo file with following contents<br> 1387 * 1. the correct tablename <br> 1388 * 2. the correct colfamily list<br> 1389 * 3. the default properties for both {@link TableDescriptor} and {@link ColumnFamilyDescriptor}<br> 1390 * @throws IOException 1391 */ 1392 private boolean fabricateTableInfo(FSTableDescriptors fstd, TableName tableName, 1393 Set<String> columns) throws IOException { 1394 if (columns ==null || columns.isEmpty()) return false; 1395 TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(tableName); 1396 for (String columnfamimly : columns) { 1397 builder.setColumnFamily(ColumnFamilyDescriptorBuilder.of(columnfamimly)); 1398 } 1399 fstd.createTableDescriptor(builder.build(), true); 1400 return true; 1401 } 1402 1403 /** 1404 * To fix the empty REGIONINFO_QUALIFIER rows from hbase:meta <br> 1405 * @throws IOException 1406 */ 1407 public void fixEmptyMetaCells() throws IOException { 1408 if (shouldFixEmptyMetaCells() && !emptyRegionInfoQualifiers.isEmpty()) { 1409 LOG.info("Trying to fix empty REGIONINFO_QUALIFIER hbase:meta rows."); 1410 for (Result region : emptyRegionInfoQualifiers) { 1411 deleteMetaRegion(region.getRow()); 1412 errors.getErrorList().remove(ERROR_CODE.EMPTY_META_CELL); 1413 } 1414 emptyRegionInfoQualifiers.clear(); 1415 } 1416 } 1417 1418 /** 1419 * To fix orphan table by creating a .tableinfo file under tableDir <br> 1420 * 1. if TableInfo is cached, to recover the .tableinfo accordingly <br> 1421 * 2. else create a default .tableinfo file with following items<br> 1422 * 2.1 the correct tablename <br> 1423 * 2.2 the correct colfamily list<br> 1424 * 2.3 the default properties for both {@link TableDescriptor} and {@link ColumnFamilyDescriptor}<br> 1425 * @throws IOException 1426 */ 1427 public void fixOrphanTables() throws IOException { 1428 if (shouldFixTableOrphans() && !orphanTableDirs.isEmpty()) { 1429 1430 List<TableName> tmpList = new ArrayList<>(orphanTableDirs.keySet().size()); 1431 tmpList.addAll(orphanTableDirs.keySet()); 1432 TableDescriptor[] htds = getTableDescriptors(tmpList); 1433 Iterator<Entry<TableName, Set<String>>> iter = 1434 orphanTableDirs.entrySet().iterator(); 1435 int j = 0; 1436 int numFailedCase = 0; 1437 FSTableDescriptors fstd = new FSTableDescriptors(getConf()); 1438 while (iter.hasNext()) { 1439 Entry<TableName, Set<String>> entry = 1440 iter.next(); 1441 TableName tableName = entry.getKey(); 1442 LOG.info("Trying to fix orphan table error: " + tableName); 1443 if (j < htds.length) { 1444 if (tableName.equals(htds[j].getTableName())) { 1445 TableDescriptor htd = htds[j]; 1446 LOG.info("fixing orphan table: " + tableName + " from cache"); 1447 fstd.createTableDescriptor(htd, true); 1448 j++; 1449 iter.remove(); 1450 } 1451 } else { 1452 if (fabricateTableInfo(fstd, tableName, entry.getValue())) { 1453 LOG.warn("fixing orphan table: " + tableName + " with a default .tableinfo file"); 1454 LOG.warn("Strongly recommend to modify the TableDescriptor if necessary for: " + tableName); 1455 iter.remove(); 1456 } else { 1457 LOG.error("Unable to create default .tableinfo for " + tableName + " while missing column family information"); 1458 numFailedCase++; 1459 } 1460 } 1461 fixes++; 1462 } 1463 1464 if (orphanTableDirs.isEmpty()) { 1465 // all orphanTableDirs are luckily recovered 1466 // re-run doFsck after recovering the .tableinfo file 1467 setShouldRerun(); 1468 LOG.warn("Strongly recommend to re-run manually hfsck after all orphanTableDirs being fixed"); 1469 } else if (numFailedCase > 0) { 1470 LOG.error("Failed to fix " + numFailedCase 1471 + " OrphanTables with default .tableinfo files"); 1472 } 1473 1474 } 1475 //cleanup the list 1476 orphanTableDirs.clear(); 1477 1478 } 1479 1480 /** 1481 * Log an appropriate message about whether or not overlapping merges are computed in parallel. 1482 */ 1483 private void logParallelMerge() { 1484 if (getConf().getBoolean("hbasefsck.overlap.merge.parallel", true)) { 1485 LOG.info("Handling overlap merges in parallel. set hbasefsck.overlap.merge.parallel to" + 1486 " false to run serially."); 1487 } else { 1488 LOG.info("Handling overlap merges serially. set hbasefsck.overlap.merge.parallel to" + 1489 " true to run in parallel."); 1490 } 1491 } 1492 1493 private SortedMap<TableName, HbckTableInfo> checkHdfsIntegrity(boolean fixHoles, 1494 boolean fixOverlaps) throws IOException { 1495 LOG.info("Checking HBase region split map from HDFS data..."); 1496 logParallelMerge(); 1497 for (HbckTableInfo tInfo : tablesInfo.values()) { 1498 TableIntegrityErrorHandler handler; 1499 if (fixHoles || fixOverlaps) { 1500 handler = tInfo.new HDFSIntegrityFixer(tInfo, errors, getConf(), 1501 fixHoles, fixOverlaps); 1502 } else { 1503 handler = tInfo.new IntegrityFixSuggester(tInfo, errors); 1504 } 1505 if (!tInfo.checkRegionChain(handler)) { 1506 // should dump info as well. 1507 errors.report("Found inconsistency in table " + tInfo.getName()); 1508 } 1509 } 1510 return tablesInfo; 1511 } 1512 1513 Path getSidelineDir() throws IOException { 1514 if (sidelineDir == null) { 1515 Path hbaseDir = CommonFSUtils.getRootDir(getConf()); 1516 Path hbckDir = new Path(hbaseDir, HConstants.HBCK_SIDELINEDIR_NAME); 1517 sidelineDir = new Path(hbckDir, hbaseDir.getName() + "-" 1518 + startMillis); 1519 } 1520 return sidelineDir; 1521 } 1522 1523 /** 1524 * Sideline a region dir (instead of deleting it) 1525 */ 1526 Path sidelineRegionDir(FileSystem fs, HbckRegionInfo hi) throws IOException { 1527 return sidelineRegionDir(fs, null, hi); 1528 } 1529 1530 /** 1531 * Sideline a region dir (instead of deleting it) 1532 * 1533 * @param parentDir if specified, the region will be sidelined to folder like 1534 * {@literal .../parentDir/<table name>/<region name>}. The purpose is to group together 1535 * similar regions sidelined, for example, those regions should be bulk loaded back later 1536 * on. If NULL, it is ignored. 1537 */ 1538 Path sidelineRegionDir(FileSystem fs, 1539 String parentDir, HbckRegionInfo hi) throws IOException { 1540 TableName tableName = hi.getTableName(); 1541 Path regionDir = hi.getHdfsRegionDir(); 1542 1543 if (!fs.exists(regionDir)) { 1544 LOG.warn("No previous " + regionDir + " exists. Continuing."); 1545 return null; 1546 } 1547 1548 Path rootDir = getSidelineDir(); 1549 if (parentDir != null) { 1550 rootDir = new Path(rootDir, parentDir); 1551 } 1552 Path sidelineTableDir= CommonFSUtils.getTableDir(rootDir, tableName); 1553 Path sidelineRegionDir = new Path(sidelineTableDir, regionDir.getName()); 1554 fs.mkdirs(sidelineRegionDir); 1555 boolean success = false; 1556 FileStatus[] cfs = fs.listStatus(regionDir); 1557 if (cfs == null) { 1558 LOG.info("Region dir is empty: " + regionDir); 1559 } else { 1560 for (FileStatus cf : cfs) { 1561 Path src = cf.getPath(); 1562 Path dst = new Path(sidelineRegionDir, src.getName()); 1563 if (fs.isFile(src)) { 1564 // simple file 1565 success = fs.rename(src, dst); 1566 if (!success) { 1567 String msg = "Unable to rename file " + src + " to " + dst; 1568 LOG.error(msg); 1569 throw new IOException(msg); 1570 } 1571 continue; 1572 } 1573 1574 // is a directory. 1575 fs.mkdirs(dst); 1576 1577 LOG.info("Sidelining files from " + src + " into containing region " + dst); 1578 // FileSystem.rename is inconsistent with directories -- if the 1579 // dst (foo/a) exists and is a dir, and the src (foo/b) is a dir, 1580 // it moves the src into the dst dir resulting in (foo/a/b). If 1581 // the dst does not exist, and the src a dir, src becomes dst. (foo/b) 1582 FileStatus[] hfiles = fs.listStatus(src); 1583 if (hfiles != null && hfiles.length > 0) { 1584 for (FileStatus hfile : hfiles) { 1585 success = fs.rename(hfile.getPath(), dst); 1586 if (!success) { 1587 String msg = "Unable to rename file " + src + " to " + dst; 1588 LOG.error(msg); 1589 throw new IOException(msg); 1590 } 1591 } 1592 } 1593 LOG.debug("Sideline directory contents:"); 1594 debugLsr(sidelineRegionDir); 1595 } 1596 } 1597 1598 LOG.info("Removing old region dir: " + regionDir); 1599 success = fs.delete(regionDir, true); 1600 if (!success) { 1601 String msg = "Unable to delete dir " + regionDir; 1602 LOG.error(msg); 1603 throw new IOException(msg); 1604 } 1605 return sidelineRegionDir; 1606 } 1607 1608 /** 1609 * Load the list of disabled tables in ZK into local set. 1610 * @throws ZooKeeperConnectionException 1611 * @throws IOException 1612 */ 1613 private void loadTableStates() 1614 throws IOException { 1615 tableStates = MetaTableAccessor.getTableStates(connection); 1616 // Add hbase:meta so this tool keeps working. In hbase2, meta is always enabled though it 1617 // has no entry in the table states. HBCK doesn't work right w/ hbase2 but just do this in 1618 // meantime. 1619 this.tableStates.put(TableName.META_TABLE_NAME, 1620 new TableState(TableName.META_TABLE_NAME, TableState.State.ENABLED)); 1621 } 1622 1623 /** 1624 * Check if the specified region's table is disabled. 1625 * @param tableName table to check status of 1626 */ 1627 boolean isTableDisabled(TableName tableName) { 1628 return tableStates.containsKey(tableName) 1629 && tableStates.get(tableName) 1630 .inStates(TableState.State.DISABLED, TableState.State.DISABLING); 1631 } 1632 1633 /** 1634 * Scan HDFS for all regions, recording their information into 1635 * regionInfoMap 1636 */ 1637 public void loadHdfsRegionDirs() throws IOException, InterruptedException { 1638 Path rootDir = CommonFSUtils.getRootDir(getConf()); 1639 FileSystem fs = rootDir.getFileSystem(getConf()); 1640 1641 // list all tables from HDFS 1642 List<FileStatus> tableDirs = Lists.newArrayList(); 1643 1644 boolean foundVersionFile = fs.exists(new Path(rootDir, HConstants.VERSION_FILE_NAME)); 1645 1646 List<Path> paths = FSUtils.getTableDirs(fs, rootDir); 1647 for (Path path : paths) { 1648 TableName tableName = CommonFSUtils.getTableName(path); 1649 if ((!checkMetaOnly && 1650 isTableIncluded(tableName)) || 1651 tableName.equals(TableName.META_TABLE_NAME)) { 1652 tableDirs.add(fs.getFileStatus(path)); 1653 } 1654 } 1655 1656 // verify that version file exists 1657 if (!foundVersionFile) { 1658 errors.reportError(ERROR_CODE.NO_VERSION_FILE, 1659 "Version file does not exist in root dir " + rootDir); 1660 if (shouldFixVersionFile()) { 1661 LOG.info("Trying to create a new " + HConstants.VERSION_FILE_NAME 1662 + " file."); 1663 setShouldRerun(); 1664 FSUtils.setVersion(fs, rootDir, getConf().getInt( 1665 HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000), getConf().getInt( 1666 HConstants.VERSION_FILE_WRITE_ATTEMPTS, 1667 HConstants.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS)); 1668 } 1669 } 1670 1671 // Avoid multithreading at table-level because already multithreaded internally at 1672 // region-level. Additionally multithreading at table-level can lead to deadlock 1673 // if there are many tables in the cluster. Since there are a limited # of threads 1674 // in the executor's thread pool and if we multithread at the table-level by putting 1675 // WorkItemHdfsDir callables into the executor, then we will have some threads in the 1676 // executor tied up solely in waiting for the tables' region-level calls to complete. 1677 // If there are enough tables then there will be no actual threads in the pool left 1678 // for the region-level callables to be serviced. 1679 for (FileStatus tableDir : tableDirs) { 1680 LOG.debug("Loading region dirs from " +tableDir.getPath()); 1681 WorkItemHdfsDir item = new WorkItemHdfsDir(fs, errors, tableDir); 1682 try { 1683 item.call(); 1684 } catch (ExecutionException e) { 1685 LOG.warn("Could not completely load table dir " + 1686 tableDir.getPath(), e.getCause()); 1687 } 1688 } 1689 errors.print(""); 1690 } 1691 1692 /** 1693 * Record the location of the hbase:meta region as found in ZooKeeper. 1694 */ 1695 private boolean recordMetaRegion() throws IOException { 1696 RegionLocations rl = connection.locateRegion(TableName.META_TABLE_NAME, 1697 HConstants.EMPTY_START_ROW, false, false); 1698 if (rl == null) { 1699 errors.reportError(ERROR_CODE.NULL_META_REGION, 1700 "META region was not found in ZooKeeper"); 1701 return false; 1702 } 1703 for (HRegionLocation metaLocation : rl.getRegionLocations()) { 1704 // Check if Meta region is valid and existing 1705 if (metaLocation == null ) { 1706 errors.reportError(ERROR_CODE.NULL_META_REGION, 1707 "META region location is null"); 1708 return false; 1709 } 1710 if (metaLocation.getRegionInfo() == null) { 1711 errors.reportError(ERROR_CODE.NULL_META_REGION, 1712 "META location regionInfo is null"); 1713 return false; 1714 } 1715 if (metaLocation.getHostname() == null) { 1716 errors.reportError(ERROR_CODE.NULL_META_REGION, 1717 "META location hostName is null"); 1718 return false; 1719 } 1720 ServerName sn = metaLocation.getServerName(); 1721 HbckRegionInfo.MetaEntry m = new HbckRegionInfo.MetaEntry(metaLocation.getRegion(), sn, 1722 EnvironmentEdgeManager.currentTime()); 1723 HbckRegionInfo hbckRegionInfo = regionInfoMap.get(metaLocation.getRegion().getEncodedName()); 1724 if (hbckRegionInfo == null) { 1725 regionInfoMap.put(metaLocation.getRegion().getEncodedName(), new HbckRegionInfo(m)); 1726 } else { 1727 hbckRegionInfo.setMetaEntry(m); 1728 } 1729 } 1730 return true; 1731 } 1732 1733 private ZKWatcher createZooKeeperWatcher() throws IOException { 1734 return new ZKWatcher(getConf(), "hbase Fsck", new Abortable() { 1735 @Override 1736 public void abort(String why, Throwable e) { 1737 LOG.error(why, e); 1738 System.exit(1); 1739 } 1740 1741 @Override 1742 public boolean isAborted() { 1743 return false; 1744 } 1745 1746 }); 1747 } 1748 1749 /** 1750 * Contacts each regionserver and fetches metadata about regions. 1751 * @param regionServerList - the list of region servers to connect to 1752 * @throws IOException if a remote or network exception occurs 1753 */ 1754 void processRegionServers(Collection<ServerName> regionServerList) 1755 throws IOException, InterruptedException { 1756 1757 List<WorkItemRegion> workItems = new ArrayList<>(regionServerList.size()); 1758 List<Future<Void>> workFutures; 1759 1760 // loop to contact each region server in parallel 1761 for (ServerName rsinfo: regionServerList) { 1762 workItems.add(new WorkItemRegion(this, rsinfo, errors, connection)); 1763 } 1764 1765 workFutures = executor.invokeAll(workItems); 1766 1767 for(int i=0; i<workFutures.size(); i++) { 1768 WorkItemRegion item = workItems.get(i); 1769 Future<Void> f = workFutures.get(i); 1770 try { 1771 f.get(); 1772 } catch(ExecutionException e) { 1773 LOG.warn("Could not process regionserver " + item.rsinfo.getHostAndPort(), 1774 e.getCause()); 1775 } 1776 } 1777 } 1778 1779 /** 1780 * Check consistency of all regions that have been found in previous phases. 1781 */ 1782 private void checkAndFixConsistency() 1783 throws IOException, KeeperException, InterruptedException { 1784 // Divide the checks in two phases. One for default/primary replicas and another 1785 // for the non-primary ones. Keeps code cleaner this way. 1786 1787 List<CheckRegionConsistencyWorkItem> workItems = new ArrayList<>(regionInfoMap.size()); 1788 for (java.util.Map.Entry<String, HbckRegionInfo> e: regionInfoMap.entrySet()) { 1789 if (e.getValue().getReplicaId() == RegionInfo.DEFAULT_REPLICA_ID) { 1790 workItems.add(new CheckRegionConsistencyWorkItem(e.getKey(), e.getValue())); 1791 } 1792 } 1793 checkRegionConsistencyConcurrently(workItems); 1794 1795 boolean prevHdfsCheck = shouldCheckHdfs(); 1796 setCheckHdfs(false); //replicas don't have any hdfs data 1797 // Run a pass over the replicas and fix any assignment issues that exist on the currently 1798 // deployed/undeployed replicas. 1799 List<CheckRegionConsistencyWorkItem> replicaWorkItems = new ArrayList<>(regionInfoMap.size()); 1800 for (java.util.Map.Entry<String, HbckRegionInfo> e: regionInfoMap.entrySet()) { 1801 if (e.getValue().getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) { 1802 replicaWorkItems.add(new CheckRegionConsistencyWorkItem(e.getKey(), e.getValue())); 1803 } 1804 } 1805 checkRegionConsistencyConcurrently(replicaWorkItems); 1806 setCheckHdfs(prevHdfsCheck); 1807 1808 // If some regions is skipped during checkRegionConsistencyConcurrently() phase, we might 1809 // not get accurate state of the hbase if continuing. The config here allows users to tune 1810 // the tolerance of number of skipped region. 1811 // TODO: evaluate the consequence to continue the hbck operation without config. 1812 int terminateThreshold = getConf().getInt("hbase.hbck.skipped.regions.limit", 0); 1813 int numOfSkippedRegions = skippedRegions.size(); 1814 if (numOfSkippedRegions > 0 && numOfSkippedRegions > terminateThreshold) { 1815 throw new IOException(numOfSkippedRegions 1816 + " region(s) could not be checked or repaired. See logs for detail."); 1817 } 1818 1819 if (shouldCheckHdfs()) { 1820 checkAndFixTableStates(); 1821 } 1822 } 1823 1824 /** 1825 * Check consistency of all regions using mulitple threads concurrently. 1826 */ 1827 private void checkRegionConsistencyConcurrently( 1828 final List<CheckRegionConsistencyWorkItem> workItems) 1829 throws IOException, KeeperException, InterruptedException { 1830 if (workItems.isEmpty()) { 1831 return; // nothing to check 1832 } 1833 1834 List<Future<Void>> workFutures = executor.invokeAll(workItems); 1835 for(Future<Void> f: workFutures) { 1836 try { 1837 f.get(); 1838 } catch(ExecutionException e1) { 1839 LOG.warn("Could not check region consistency " , e1.getCause()); 1840 if (e1.getCause() instanceof IOException) { 1841 throw (IOException)e1.getCause(); 1842 } else if (e1.getCause() instanceof KeeperException) { 1843 throw (KeeperException)e1.getCause(); 1844 } else if (e1.getCause() instanceof InterruptedException) { 1845 throw (InterruptedException)e1.getCause(); 1846 } else { 1847 throw new IOException(e1.getCause()); 1848 } 1849 } 1850 } 1851 } 1852 1853 class CheckRegionConsistencyWorkItem implements Callable<Void> { 1854 private final String key; 1855 private final HbckRegionInfo hbi; 1856 1857 CheckRegionConsistencyWorkItem(String key, HbckRegionInfo hbi) { 1858 this.key = key; 1859 this.hbi = hbi; 1860 } 1861 1862 @Override 1863 public synchronized Void call() throws Exception { 1864 try { 1865 checkRegionConsistency(key, hbi); 1866 } catch (Exception e) { 1867 // If the region is non-META region, skip this region and send warning/error message; if 1868 // the region is META region, we should not continue. 1869 LOG.warn("Unable to complete check or repair the region '" + hbi.getRegionNameAsString() 1870 + "'.", e); 1871 if (hbi.getHdfsHRI().isMetaRegion()) { 1872 throw e; 1873 } 1874 LOG.warn("Skip region '" + hbi.getRegionNameAsString() + "'"); 1875 addSkippedRegion(hbi); 1876 } 1877 return null; 1878 } 1879 } 1880 1881 private void addSkippedRegion(final HbckRegionInfo hbi) { 1882 Set<String> skippedRegionNames = skippedRegions.get(hbi.getTableName()); 1883 if (skippedRegionNames == null) { 1884 skippedRegionNames = new HashSet<>(); 1885 } 1886 skippedRegionNames.add(hbi.getRegionNameAsString()); 1887 skippedRegions.put(hbi.getTableName(), skippedRegionNames); 1888 } 1889 1890 /** 1891 * Check and fix table states, assumes full info available: 1892 * - tableInfos 1893 * - empty tables loaded 1894 */ 1895 private void checkAndFixTableStates() throws IOException { 1896 // first check dangling states 1897 for (Entry<TableName, TableState> entry : tableStates.entrySet()) { 1898 TableName tableName = entry.getKey(); 1899 TableState tableState = entry.getValue(); 1900 HbckTableInfo tableInfo = tablesInfo.get(tableName); 1901 if (isTableIncluded(tableName) 1902 && !tableName.isSystemTable() 1903 && tableInfo == null) { 1904 if (fixMeta) { 1905 MetaTableAccessor.deleteTableState(connection, tableName); 1906 TableState state = MetaTableAccessor.getTableState(connection, tableName); 1907 if (state != null) { 1908 errors.reportError(ERROR_CODE.ORPHAN_TABLE_STATE, 1909 tableName + " unable to delete dangling table state " + tableState); 1910 } 1911 } else if (!checkMetaOnly) { 1912 // dangling table state in meta if checkMetaOnly is false. If checkMetaOnly is 1913 // true, tableInfo will be null as tablesInfo are not polulated for all tables from hdfs 1914 errors.reportError(ERROR_CODE.ORPHAN_TABLE_STATE, 1915 tableName + " has dangling table state " + tableState); 1916 } 1917 } 1918 } 1919 // check that all tables have states 1920 for (TableName tableName : tablesInfo.keySet()) { 1921 if (isTableIncluded(tableName) && !tableStates.containsKey(tableName)) { 1922 if (fixMeta) { 1923 MetaTableAccessor.updateTableState(connection, tableName, TableState.State.ENABLED); 1924 TableState newState = MetaTableAccessor.getTableState(connection, tableName); 1925 if (newState == null) { 1926 errors.reportError(ERROR_CODE.NO_TABLE_STATE, 1927 "Unable to change state for table " + tableName + " in meta "); 1928 } 1929 } else { 1930 errors.reportError(ERROR_CODE.NO_TABLE_STATE, 1931 tableName + " has no state in meta "); 1932 } 1933 } 1934 } 1935 } 1936 1937 private void preCheckPermission() throws IOException { 1938 if (shouldIgnorePreCheckPermission()) { 1939 return; 1940 } 1941 1942 Path hbaseDir = CommonFSUtils.getRootDir(getConf()); 1943 FileSystem fs = hbaseDir.getFileSystem(getConf()); 1944 UserProvider userProvider = UserProvider.instantiate(getConf()); 1945 UserGroupInformation ugi = userProvider.getCurrent().getUGI(); 1946 FileStatus[] files = fs.listStatus(hbaseDir); 1947 for (FileStatus file : files) { 1948 try { 1949 fs.access(file.getPath(), FsAction.WRITE); 1950 } catch (AccessControlException ace) { 1951 LOG.warn("Got AccessDeniedException when preCheckPermission ", ace); 1952 errors.reportError(ERROR_CODE.WRONG_USAGE, "Current user " + ugi.getUserName() 1953 + " does not have write perms to " + file.getPath() 1954 + ". Please rerun hbck as hdfs user " + file.getOwner()); 1955 throw ace; 1956 } 1957 } 1958 } 1959 1960 /** 1961 * Deletes region from meta table 1962 */ 1963 private void deleteMetaRegion(HbckRegionInfo hi) throws IOException { 1964 deleteMetaRegion(hi.getMetaEntry().getRegionName()); 1965 } 1966 1967 /** 1968 * Deletes region from meta table 1969 */ 1970 private void deleteMetaRegion(byte[] metaKey) throws IOException { 1971 Delete d = new Delete(metaKey); 1972 meta.delete(d); 1973 LOG.info("Deleted " + Bytes.toString(metaKey) + " from META" ); 1974 } 1975 1976 /** 1977 * Reset the split parent region info in meta table 1978 */ 1979 private void resetSplitParent(HbckRegionInfo hi) throws IOException { 1980 RowMutations mutations = new RowMutations(hi.getMetaEntry().getRegionName()); 1981 Delete d = new Delete(hi.getMetaEntry().getRegionName()); 1982 d.addColumn(HConstants.CATALOG_FAMILY, HConstants.SPLITA_QUALIFIER); 1983 d.addColumn(HConstants.CATALOG_FAMILY, HConstants.SPLITB_QUALIFIER); 1984 mutations.add(d); 1985 1986 RegionInfo hri = RegionInfoBuilder.newBuilder(hi.getMetaEntry()) 1987 .setOffline(false) 1988 .setSplit(false) 1989 .build(); 1990 Put p = MetaTableAccessor.makePutFromRegionInfo(hri, EnvironmentEdgeManager.currentTime()); 1991 mutations.add(p); 1992 1993 meta.mutateRow(mutations); 1994 LOG.info("Reset split parent " + hi.getMetaEntry().getRegionNameAsString() + " in META"); 1995 } 1996 1997 /** 1998 * This backwards-compatibility wrapper for permanently offlining a region 1999 * that should not be alive. If the region server does not support the 2000 * "offline" method, it will use the closest unassign method instead. This 2001 * will basically work until one attempts to disable or delete the affected 2002 * table. The problem has to do with in-memory only master state, so 2003 * restarting the HMaster or failing over to another should fix this. 2004 */ 2005 void offline(byte[] regionName) throws IOException { 2006 String regionString = Bytes.toStringBinary(regionName); 2007 if (!rsSupportsOffline) { 2008 LOG.warn( 2009 "Using unassign region " + regionString + " instead of using offline method, you should" + 2010 " restart HMaster after these repairs"); 2011 admin.unassign(regionName, true); 2012 return; 2013 } 2014 2015 // first time we assume the rs's supports #offline. 2016 try { 2017 LOG.info("Offlining region " + regionString); 2018 admin.offline(regionName); 2019 } catch (IOException ioe) { 2020 String notFoundMsg = "java.lang.NoSuchMethodException: " + 2021 "org.apache.hadoop.hbase.master.HMaster.offline([B)"; 2022 if (ioe.getMessage().contains(notFoundMsg)) { 2023 LOG.warn("Using unassign region " + regionString + 2024 " instead of using offline method, you should" + 2025 " restart HMaster after these repairs"); 2026 rsSupportsOffline = false; // in the future just use unassign 2027 admin.unassign(regionName, true); 2028 return; 2029 } 2030 throw ioe; 2031 } 2032 } 2033 2034 /** 2035 * Attempts to undeploy a region from a region server based in information in 2036 * META. Any operations that modify the file system should make sure that 2037 * its corresponding region is not deployed to prevent data races. 2038 * 2039 * A separate call is required to update the master in-memory region state 2040 * kept in the AssignementManager. Because disable uses this state instead of 2041 * that found in META, we can't seem to cleanly disable/delete tables that 2042 * have been hbck fixed. When used on a version of HBase that does not have 2043 * the offline ipc call exposed on the master (<0.90.5, <0.92.0) a master 2044 * restart or failover may be required. 2045 */ 2046 void closeRegion(HbckRegionInfo hi) throws IOException, InterruptedException { 2047 if (hi.getMetaEntry() == null && hi.getHdfsEntry() == null) { 2048 undeployRegions(hi); 2049 return; 2050 } 2051 2052 // get assignment info and hregioninfo from meta. 2053 Get get = new Get(hi.getRegionName()); 2054 get.addColumn(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER); 2055 get.addColumn(HConstants.CATALOG_FAMILY, HConstants.SERVER_QUALIFIER); 2056 get.addColumn(HConstants.CATALOG_FAMILY, HConstants.STARTCODE_QUALIFIER); 2057 // also get the locations of the replicas to close if the primary region is being closed 2058 if (hi.getReplicaId() == RegionInfo.DEFAULT_REPLICA_ID) { 2059 int numReplicas = admin.getTableDescriptor(hi.getTableName()).getRegionReplication(); 2060 for (int i = 0; i < numReplicas; i++) { 2061 get.addColumn(HConstants.CATALOG_FAMILY, MetaTableAccessor.getServerColumn(i)); 2062 get.addColumn(HConstants.CATALOG_FAMILY, MetaTableAccessor.getStartCodeColumn(i)); 2063 } 2064 } 2065 Result r = meta.get(get); 2066 RegionLocations rl = MetaTableAccessor.getRegionLocations(r); 2067 if (rl == null) { 2068 LOG.warn("Unable to close region " + hi.getRegionNameAsString() + 2069 " since meta does not have handle to reach it"); 2070 return; 2071 } 2072 for (HRegionLocation h : rl.getRegionLocations()) { 2073 ServerName serverName = h.getServerName(); 2074 if (serverName == null) { 2075 errors.reportError("Unable to close region " 2076 + hi.getRegionNameAsString() + " because meta does not " 2077 + "have handle to reach it."); 2078 continue; 2079 } 2080 RegionInfo hri = h.getRegionInfo(); 2081 if (hri == null) { 2082 LOG.warn("Unable to close region " + hi.getRegionNameAsString() 2083 + " because hbase:meta had invalid or missing " 2084 + HConstants.CATALOG_FAMILY_STR + ":" 2085 + Bytes.toString(HConstants.REGIONINFO_QUALIFIER) 2086 + " qualifier value."); 2087 continue; 2088 } 2089 // close the region -- close files and remove assignment 2090 HBaseFsckRepair.closeRegionSilentlyAndWait(connection, serverName, hri); 2091 } 2092 } 2093 2094 private void undeployRegions(HbckRegionInfo hi) throws IOException, InterruptedException { 2095 undeployRegionsForHbi(hi); 2096 // undeploy replicas of the region (but only if the method is invoked for the primary) 2097 if (hi.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) { 2098 return; 2099 } 2100 int numReplicas = admin.getDescriptor(hi.getTableName()).getRegionReplication(); 2101 for (int i = 1; i < numReplicas; i++) { 2102 if (hi.getPrimaryHRIForDeployedReplica() == null) continue; 2103 RegionInfo hri = RegionReplicaUtil.getRegionInfoForReplica( 2104 hi.getPrimaryHRIForDeployedReplica(), i); 2105 HbckRegionInfo h = regionInfoMap.get(hri.getEncodedName()); 2106 if (h != null) { 2107 undeployRegionsForHbi(h); 2108 //set skip checks; we undeployed it, and we don't want to evaluate this anymore 2109 //in consistency checks 2110 h.setSkipChecks(true); 2111 } 2112 } 2113 } 2114 2115 private void undeployRegionsForHbi(HbckRegionInfo hi) throws IOException, InterruptedException { 2116 for (HbckRegionInfo.OnlineEntry rse : hi.getOnlineEntries()) { 2117 LOG.debug("Undeploy region " + rse.getRegionInfo() + " from " + rse.getServerName()); 2118 try { 2119 HBaseFsckRepair 2120 .closeRegionSilentlyAndWait(connection, rse.getServerName(), rse.getRegionInfo()); 2121 offline(rse.getRegionInfo().getRegionName()); 2122 } catch (IOException ioe) { 2123 LOG.warn("Got exception when attempting to offline region " 2124 + Bytes.toString(rse.getRegionInfo().getRegionName()), ioe); 2125 } 2126 } 2127 } 2128 2129 private void tryAssignmentRepair(HbckRegionInfo hbi, String msg) throws IOException, 2130 KeeperException, InterruptedException { 2131 // If we are trying to fix the errors 2132 if (shouldFixAssignments()) { 2133 errors.print(msg); 2134 undeployRegions(hbi); 2135 setShouldRerun(); 2136 RegionInfo hri = hbi.getHdfsHRI(); 2137 if (hri == null) { 2138 hri = hbi.getMetaEntry(); 2139 } 2140 HBaseFsckRepair.fixUnassigned(admin, hri); 2141 HBaseFsckRepair.waitUntilAssigned(admin, hri); 2142 2143 // also assign replicas if needed (do it only when this call operates on a primary replica) 2144 if (hbi.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) return; 2145 int replicationCount = admin.getTableDescriptor(hri.getTable()).getRegionReplication(); 2146 for (int i = 1; i < replicationCount; i++) { 2147 hri = RegionReplicaUtil.getRegionInfoForReplica(hri, i); 2148 HbckRegionInfo h = regionInfoMap.get(hri.getEncodedName()); 2149 if (h != null) { 2150 undeployRegions(h); 2151 //set skip checks; we undeploy & deploy it; we don't want to evaluate this hbi anymore 2152 //in consistency checks 2153 h.setSkipChecks(true); 2154 } 2155 HBaseFsckRepair.fixUnassigned(admin, hri); 2156 HBaseFsckRepair.waitUntilAssigned(admin, hri); 2157 } 2158 2159 } 2160 } 2161 2162 /** 2163 * Check a single region for consistency and correct deployment. 2164 */ 2165 private void checkRegionConsistency(final String key, final HbckRegionInfo hbi) 2166 throws IOException, KeeperException, InterruptedException { 2167 2168 if (hbi.isSkipChecks()) return; 2169 String descriptiveName = hbi.toString(); 2170 boolean inMeta = hbi.getMetaEntry() != null; 2171 // In case not checking HDFS, assume the region is on HDFS 2172 boolean inHdfs = !shouldCheckHdfs() || hbi.getHdfsRegionDir() != null; 2173 boolean hasMetaAssignment = inMeta && hbi.getMetaEntry().regionServer != null; 2174 boolean isDeployed = !hbi.getDeployedOn().isEmpty(); 2175 boolean isMultiplyDeployed = hbi.getDeployedOn().size() > 1; 2176 boolean deploymentMatchesMeta = 2177 hasMetaAssignment && isDeployed && !isMultiplyDeployed && 2178 hbi.getMetaEntry().regionServer.equals(hbi.getDeployedOn().get(0)); 2179 boolean splitParent = 2180 inMeta && hbi.getMetaEntry().isSplit() && hbi.getMetaEntry().isOffline(); 2181 boolean shouldBeDeployed = inMeta && !isTableDisabled(hbi.getMetaEntry().getTable()); 2182 boolean recentlyModified = inHdfs && 2183 hbi.getModTime() + timelag > EnvironmentEdgeManager.currentTime(); 2184 2185 // ========== First the healthy cases ============= 2186 if (hbi.containsOnlyHdfsEdits()) { 2187 return; 2188 } 2189 if (inMeta && inHdfs && isDeployed && deploymentMatchesMeta && shouldBeDeployed) { 2190 return; 2191 } else if (inMeta && inHdfs && !shouldBeDeployed && !isDeployed) { 2192 LOG.info("Region " + descriptiveName + " is in META, and in a disabled " + 2193 "tabled that is not deployed"); 2194 return; 2195 } else if (recentlyModified) { 2196 LOG.warn("Region " + descriptiveName + " was recently modified -- skipping"); 2197 return; 2198 } 2199 // ========== Cases where the region is not in hbase:meta ============= 2200 else if (!inMeta && !inHdfs && !isDeployed) { 2201 // We shouldn't have record of this region at all then! 2202 assert false : "Entry for region with no data"; 2203 } else if (!inMeta && !inHdfs && isDeployed) { 2204 errors.reportError(ERROR_CODE.NOT_IN_META_HDFS, "Region " 2205 + descriptiveName + ", key=" + key + ", not on HDFS or in hbase:meta but " + 2206 "deployed on " + Joiner.on(", ").join(hbi.getDeployedOn())); 2207 if (shouldFixAssignments()) { 2208 undeployRegions(hbi); 2209 } 2210 2211 } else if (!inMeta && inHdfs && !isDeployed) { 2212 if (hbi.isMerged()) { 2213 // This region has already been merged, the remaining hdfs file will be 2214 // cleaned by CatalogJanitor later 2215 hbi.setSkipChecks(true); 2216 LOG.info("Region " + descriptiveName 2217 + " got merge recently, its file(s) will be cleaned by CatalogJanitor later"); 2218 return; 2219 } 2220 errors.reportError(ERROR_CODE.NOT_IN_META_OR_DEPLOYED, "Region " 2221 + descriptiveName + " on HDFS, but not listed in hbase:meta " + 2222 "or deployed on any region server"); 2223 // restore region consistency of an adopted orphan 2224 if (shouldFixMeta()) { 2225 if (!hbi.isHdfsRegioninfoPresent()) { 2226 LOG.error("Region " + hbi.getHdfsHRI() + " could have been repaired" 2227 + " in table integrity repair phase if -fixHdfsOrphans was" + 2228 " used."); 2229 return; 2230 } 2231 2232 RegionInfo hri = hbi.getHdfsHRI(); 2233 HbckTableInfo tableInfo = tablesInfo.get(hri.getTable()); 2234 2235 for (RegionInfo region : tableInfo.getRegionsFromMeta(this.regionInfoMap)) { 2236 if (Bytes.compareTo(region.getStartKey(), hri.getStartKey()) <= 0 2237 && (region.getEndKey().length == 0 || Bytes.compareTo(region.getEndKey(), 2238 hri.getEndKey()) >= 0) 2239 && Bytes.compareTo(region.getStartKey(), hri.getEndKey()) <= 0) { 2240 if(region.isSplit() || region.isOffline()) continue; 2241 Path regionDir = hbi.getHdfsRegionDir(); 2242 FileSystem fs = regionDir.getFileSystem(getConf()); 2243 List<Path> familyDirs = FSUtils.getFamilyDirs(fs, regionDir); 2244 for (Path familyDir : familyDirs) { 2245 List<Path> referenceFilePaths = FSUtils.getReferenceFilePaths(fs, familyDir); 2246 for (Path referenceFilePath : referenceFilePaths) { 2247 Path parentRegionDir = 2248 StoreFileInfo.getReferredToFile(referenceFilePath).getParent().getParent(); 2249 if (parentRegionDir.toString().endsWith(region.getEncodedName())) { 2250 LOG.warn(hri + " start and stop keys are in the range of " + region 2251 + ". The region might not be cleaned up from hdfs when region " + region 2252 + " split failed. Hence deleting from hdfs."); 2253 HRegionFileSystem.deleteRegionFromFileSystem(getConf(), fs, 2254 regionDir.getParent(), hri); 2255 return; 2256 } 2257 } 2258 } 2259 } 2260 } 2261 LOG.info("Patching hbase:meta with .regioninfo: " + hbi.getHdfsHRI()); 2262 int numReplicas = admin.getTableDescriptor(hbi.getTableName()).getRegionReplication(); 2263 HBaseFsckRepair.fixMetaHoleOnlineAndAddReplicas(getConf(), hbi.getHdfsHRI(), 2264 admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS)) 2265 .getLiveServerMetrics().keySet(), numReplicas); 2266 2267 tryAssignmentRepair(hbi, "Trying to reassign region..."); 2268 } 2269 2270 } else if (!inMeta && inHdfs && isDeployed) { 2271 errors.reportError(ERROR_CODE.NOT_IN_META, "Region " + descriptiveName 2272 + " not in META, but deployed on " + Joiner.on(", ").join(hbi.getDeployedOn())); 2273 debugLsr(hbi.getHdfsRegionDir()); 2274 if (hbi.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) { 2275 // for replicas, this means that we should undeploy the region (we would have 2276 // gone over the primaries and fixed meta holes in first phase under 2277 // checkAndFixConsistency; we shouldn't get the condition !inMeta at 2278 // this stage unless unwanted replica) 2279 if (shouldFixAssignments()) { 2280 undeployRegionsForHbi(hbi); 2281 } 2282 } 2283 if (shouldFixMeta() && hbi.getReplicaId() == RegionInfo.DEFAULT_REPLICA_ID) { 2284 if (!hbi.isHdfsRegioninfoPresent()) { 2285 LOG.error("This should have been repaired in table integrity repair phase"); 2286 return; 2287 } 2288 2289 LOG.info("Patching hbase:meta with with .regioninfo: " + hbi.getHdfsHRI()); 2290 int numReplicas = admin.getTableDescriptor(hbi.getTableName()).getRegionReplication(); 2291 HBaseFsckRepair.fixMetaHoleOnlineAndAddReplicas(getConf(), hbi.getHdfsHRI(), 2292 admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS)) 2293 .getLiveServerMetrics().keySet(), numReplicas); 2294 tryAssignmentRepair(hbi, "Trying to fix unassigned region..."); 2295 } 2296 2297 // ========== Cases where the region is in hbase:meta ============= 2298 } else if (inMeta && inHdfs && !isDeployed && splitParent) { 2299 // check whether this is an actual error, or just transient state where parent 2300 // is not cleaned 2301 if (hbi.getMetaEntry().splitA != null && hbi.getMetaEntry().splitB != null) { 2302 // check that split daughters are there 2303 HbckRegionInfo infoA = this.regionInfoMap.get(hbi.getMetaEntry().splitA.getEncodedName()); 2304 HbckRegionInfo infoB = this.regionInfoMap.get(hbi.getMetaEntry().splitB.getEncodedName()); 2305 if (infoA != null && infoB != null) { 2306 // we already processed or will process daughters. Move on, nothing to see here. 2307 hbi.setSkipChecks(true); 2308 return; 2309 } 2310 } 2311 2312 // For Replica region, we need to do a similar check. If replica is not split successfully, 2313 // error is going to be reported against primary daughter region. 2314 if (hbi.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) { 2315 LOG.info("Region " + descriptiveName + " is a split parent in META, in HDFS, " 2316 + "and not deployed on any region server. This may be transient."); 2317 hbi.setSkipChecks(true); 2318 return; 2319 } 2320 2321 errors.reportError(ERROR_CODE.LINGERING_SPLIT_PARENT, "Region " 2322 + descriptiveName + " is a split parent in META, in HDFS, " 2323 + "and not deployed on any region server. This could be transient, " 2324 + "consider to run the catalog janitor first!"); 2325 if (shouldFixSplitParents()) { 2326 setShouldRerun(); 2327 resetSplitParent(hbi); 2328 } 2329 } else if (inMeta && !inHdfs && !isDeployed) { 2330 errors.reportError(ERROR_CODE.NOT_IN_HDFS_OR_DEPLOYED, "Region " 2331 + descriptiveName + " found in META, but not in HDFS " 2332 + "or deployed on any region server."); 2333 if (shouldFixMeta()) { 2334 deleteMetaRegion(hbi); 2335 } 2336 } else if (inMeta && !inHdfs && isDeployed) { 2337 errors.reportError(ERROR_CODE.NOT_IN_HDFS, "Region " + descriptiveName 2338 + " found in META, but not in HDFS, " + 2339 "and deployed on " + Joiner.on(", ").join(hbi.getDeployedOn())); 2340 // We treat HDFS as ground truth. Any information in meta is transient 2341 // and equivalent data can be regenerated. So, lets unassign and remove 2342 // these problems from META. 2343 if (shouldFixAssignments()) { 2344 errors.print("Trying to fix unassigned region..."); 2345 undeployRegions(hbi); 2346 } 2347 if (shouldFixMeta()) { 2348 // wait for it to complete 2349 deleteMetaRegion(hbi); 2350 } 2351 } else if (inMeta && inHdfs && !isDeployed && shouldBeDeployed) { 2352 errors.reportError(ERROR_CODE.NOT_DEPLOYED, "Region " + descriptiveName 2353 + " not deployed on any region server."); 2354 tryAssignmentRepair(hbi, "Trying to fix unassigned region..."); 2355 } else if (inMeta && inHdfs && isDeployed && !shouldBeDeployed) { 2356 errors.reportError(ERROR_CODE.SHOULD_NOT_BE_DEPLOYED, 2357 "Region " + descriptiveName + " should not be deployed according " + 2358 "to META, but is deployed on " + Joiner.on(", ").join(hbi.getDeployedOn())); 2359 if (shouldFixAssignments()) { 2360 errors.print("Trying to close the region " + descriptiveName); 2361 setShouldRerun(); 2362 HBaseFsckRepair.fixMultiAssignment(connection, hbi.getMetaEntry(), hbi.getDeployedOn()); 2363 } 2364 } else if (inMeta && inHdfs && isMultiplyDeployed) { 2365 errors.reportError(ERROR_CODE.MULTI_DEPLOYED, "Region " + descriptiveName 2366 + " is listed in hbase:meta on region server " + hbi.getMetaEntry().regionServer 2367 + " but is multiply assigned to region servers " + 2368 Joiner.on(", ").join(hbi.getDeployedOn())); 2369 // If we are trying to fix the errors 2370 if (shouldFixAssignments()) { 2371 errors.print("Trying to fix assignment error..."); 2372 setShouldRerun(); 2373 HBaseFsckRepair.fixMultiAssignment(connection, hbi.getMetaEntry(), hbi.getDeployedOn()); 2374 } 2375 } else if (inMeta && inHdfs && isDeployed && !deploymentMatchesMeta) { 2376 errors.reportError(ERROR_CODE.SERVER_DOES_NOT_MATCH_META, "Region " 2377 + descriptiveName + " listed in hbase:meta on region server " + 2378 hbi.getMetaEntry().regionServer + " but found on region server " + 2379 hbi.getDeployedOn().get(0)); 2380 // If we are trying to fix the errors 2381 if (shouldFixAssignments()) { 2382 errors.print("Trying to fix assignment error..."); 2383 setShouldRerun(); 2384 HBaseFsckRepair.fixMultiAssignment(connection, hbi.getMetaEntry(), hbi.getDeployedOn()); 2385 HBaseFsckRepair.waitUntilAssigned(admin, hbi.getHdfsHRI()); 2386 } 2387 } else { 2388 errors.reportError(ERROR_CODE.UNKNOWN, "Region " + descriptiveName + 2389 " is in an unforeseen state:" + 2390 " inMeta=" + inMeta + 2391 " inHdfs=" + inHdfs + 2392 " isDeployed=" + isDeployed + 2393 " isMultiplyDeployed=" + isMultiplyDeployed + 2394 " deploymentMatchesMeta=" + deploymentMatchesMeta + 2395 " shouldBeDeployed=" + shouldBeDeployed); 2396 } 2397 } 2398 2399 /** 2400 * Checks tables integrity. Goes over all regions and scans the tables. 2401 * Collects all the pieces for each table and checks if there are missing, 2402 * repeated or overlapping ones. 2403 * @throws IOException 2404 */ 2405 SortedMap<TableName, HbckTableInfo> checkIntegrity() throws IOException { 2406 tablesInfo = new TreeMap<>(); 2407 LOG.debug("There are " + regionInfoMap.size() + " region info entries"); 2408 for (HbckRegionInfo hbi : regionInfoMap.values()) { 2409 // Check only valid, working regions 2410 if (hbi.getMetaEntry() == null) { 2411 // this assumes that consistency check has run loadMetaEntry 2412 Path p = hbi.getHdfsRegionDir(); 2413 if (p == null) { 2414 errors.report("No regioninfo in Meta or HDFS. " + hbi); 2415 } 2416 2417 // TODO test. 2418 continue; 2419 } 2420 if (hbi.getMetaEntry().regionServer == null) { 2421 errors.detail("Skipping region because no region server: " + hbi); 2422 continue; 2423 } 2424 if (hbi.getMetaEntry().isOffline()) { 2425 errors.detail("Skipping region because it is offline: " + hbi); 2426 continue; 2427 } 2428 if (hbi.containsOnlyHdfsEdits()) { 2429 errors.detail("Skipping region because it only contains edits" + hbi); 2430 continue; 2431 } 2432 2433 // Missing regionDir or over-deployment is checked elsewhere. Include 2434 // these cases in modTInfo, so we can evaluate those regions as part of 2435 // the region chain in META 2436 //if (hbi.foundRegionDir == null) continue; 2437 //if (hbi.deployedOn.size() != 1) continue; 2438 if (hbi.getDeployedOn().isEmpty()) { 2439 continue; 2440 } 2441 2442 // We should be safe here 2443 TableName tableName = hbi.getMetaEntry().getTable(); 2444 HbckTableInfo modTInfo = tablesInfo.get(tableName); 2445 if (modTInfo == null) { 2446 modTInfo = new HbckTableInfo(tableName, this); 2447 } 2448 for (ServerName server : hbi.getDeployedOn()) { 2449 modTInfo.addServer(server); 2450 } 2451 2452 if (!hbi.isSkipChecks()) { 2453 modTInfo.addRegionInfo(hbi); 2454 } 2455 2456 tablesInfo.put(tableName, modTInfo); 2457 } 2458 2459 loadTableInfosForTablesWithNoRegion(); 2460 2461 logParallelMerge(); 2462 for (HbckTableInfo tInfo : tablesInfo.values()) { 2463 TableIntegrityErrorHandler handler = tInfo.new IntegrityFixSuggester(tInfo, errors); 2464 if (!tInfo.checkRegionChain(handler)) { 2465 errors.report("Found inconsistency in table " + tInfo.getName()); 2466 } 2467 } 2468 return tablesInfo; 2469 } 2470 2471 /** Loads table info's for tables that may not have been included, since there are no 2472 * regions reported for the table, but table dir is there in hdfs 2473 */ 2474 private void loadTableInfosForTablesWithNoRegion() throws IOException { 2475 Map<String, TableDescriptor> allTables = new FSTableDescriptors(getConf()).getAll(); 2476 for (TableDescriptor htd : allTables.values()) { 2477 if (checkMetaOnly && !htd.isMetaTable()) { 2478 continue; 2479 } 2480 2481 TableName tableName = htd.getTableName(); 2482 if (isTableIncluded(tableName) && !tablesInfo.containsKey(tableName)) { 2483 HbckTableInfo tableInfo = new HbckTableInfo(tableName, this); 2484 tableInfo.htds.add(htd); 2485 tablesInfo.put(htd.getTableName(), tableInfo); 2486 } 2487 } 2488 } 2489 2490 /** 2491 * Merge hdfs data by moving from contained HbckRegionInfo into targetRegionDir. 2492 * @return number of file move fixes done to merge regions. 2493 */ 2494 public int mergeRegionDirs(Path targetRegionDir, HbckRegionInfo contained) throws IOException { 2495 int fileMoves = 0; 2496 String thread = Thread.currentThread().getName(); 2497 LOG.debug("[" + thread + "] Contained region dir after close and pause"); 2498 debugLsr(contained.getHdfsRegionDir()); 2499 2500 // rename the contained into the container. 2501 FileSystem fs = targetRegionDir.getFileSystem(getConf()); 2502 FileStatus[] dirs = null; 2503 try { 2504 dirs = fs.listStatus(contained.getHdfsRegionDir()); 2505 } catch (FileNotFoundException fnfe) { 2506 // region we are attempting to merge in is not present! Since this is a merge, there is 2507 // no harm skipping this region if it does not exist. 2508 if (!fs.exists(contained.getHdfsRegionDir())) { 2509 LOG.warn("[" + thread + "] HDFS region dir " + contained.getHdfsRegionDir() 2510 + " is missing. Assuming already sidelined or moved."); 2511 } else { 2512 sidelineRegionDir(fs, contained); 2513 } 2514 return fileMoves; 2515 } 2516 2517 if (dirs == null) { 2518 if (!fs.exists(contained.getHdfsRegionDir())) { 2519 LOG.warn("[" + thread + "] HDFS region dir " + contained.getHdfsRegionDir() 2520 + " already sidelined."); 2521 } else { 2522 sidelineRegionDir(fs, contained); 2523 } 2524 return fileMoves; 2525 } 2526 2527 for (FileStatus cf : dirs) { 2528 Path src = cf.getPath(); 2529 Path dst = new Path(targetRegionDir, src.getName()); 2530 2531 if (src.getName().equals(HRegionFileSystem.REGION_INFO_FILE)) { 2532 // do not copy the old .regioninfo file. 2533 continue; 2534 } 2535 2536 if (src.getName().equals(HConstants.HREGION_OLDLOGDIR_NAME)) { 2537 // do not copy the .oldlogs files 2538 continue; 2539 } 2540 2541 LOG.info("[" + thread + "] Moving files from " + src + " into containing region " + dst); 2542 // FileSystem.rename is inconsistent with directories -- if the 2543 // dst (foo/a) exists and is a dir, and the src (foo/b) is a dir, 2544 // it moves the src into the dst dir resulting in (foo/a/b). If 2545 // the dst does not exist, and the src a dir, src becomes dst. (foo/b) 2546 for (FileStatus hfile : fs.listStatus(src)) { 2547 boolean success = fs.rename(hfile.getPath(), dst); 2548 if (success) { 2549 fileMoves++; 2550 } 2551 } 2552 LOG.debug("[" + thread + "] Sideline directory contents:"); 2553 debugLsr(targetRegionDir); 2554 } 2555 2556 // if all success. 2557 sidelineRegionDir(fs, contained); 2558 LOG.info("[" + thread + "] Sidelined region dir "+ contained.getHdfsRegionDir() + " into " + 2559 getSidelineDir()); 2560 debugLsr(contained.getHdfsRegionDir()); 2561 2562 return fileMoves; 2563 } 2564 2565 2566 static class WorkItemOverlapMerge implements Callable<Void> { 2567 private TableIntegrityErrorHandler handler; 2568 Collection<HbckRegionInfo> overlapgroup; 2569 2570 WorkItemOverlapMerge(Collection<HbckRegionInfo> overlapgroup, 2571 TableIntegrityErrorHandler handler) { 2572 this.handler = handler; 2573 this.overlapgroup = overlapgroup; 2574 } 2575 2576 @Override 2577 public Void call() throws Exception { 2578 handler.handleOverlapGroup(overlapgroup); 2579 return null; 2580 } 2581 }; 2582 2583 /** 2584 * Return a list of user-space table names whose metadata have not been 2585 * modified in the last few milliseconds specified by timelag 2586 * if any of the REGIONINFO_QUALIFIER, SERVER_QUALIFIER, STARTCODE_QUALIFIER, 2587 * SPLITA_QUALIFIER, SPLITB_QUALIFIER have not changed in the last 2588 * milliseconds specified by timelag, then the table is a candidate to be returned. 2589 * @return tables that have not been modified recently 2590 * @throws IOException if an error is encountered 2591 */ 2592 TableDescriptor[] getTables(AtomicInteger numSkipped) { 2593 List<TableName> tableNames = new ArrayList<>(); 2594 long now = EnvironmentEdgeManager.currentTime(); 2595 2596 for (HbckRegionInfo hbi : regionInfoMap.values()) { 2597 HbckRegionInfo.MetaEntry info = hbi.getMetaEntry(); 2598 2599 // if the start key is zero, then we have found the first region of a table. 2600 // pick only those tables that were not modified in the last few milliseconds. 2601 if (info != null && info.getStartKey().length == 0 && !info.isMetaRegion()) { 2602 if (info.modTime + timelag < now) { 2603 tableNames.add(info.getTable()); 2604 } else { 2605 numSkipped.incrementAndGet(); // one more in-flux table 2606 } 2607 } 2608 } 2609 return getTableDescriptors(tableNames); 2610 } 2611 2612 TableDescriptor[] getTableDescriptors(List<TableName> tableNames) { 2613 LOG.info("getTableDescriptors == tableNames => " + tableNames); 2614 try (Connection conn = ConnectionFactory.createConnection(getConf()); 2615 Admin admin = conn.getAdmin()) { 2616 List<TableDescriptor> tds = admin.listTableDescriptors(tableNames); 2617 return tds.toArray(new TableDescriptor[tds.size()]); 2618 } catch (IOException e) { 2619 LOG.debug("Exception getting table descriptors", e); 2620 } 2621 return new TableDescriptor[0]; 2622 } 2623 2624 /** 2625 * Gets the entry in regionInfo corresponding to the the given encoded 2626 * region name. If the region has not been seen yet, a new entry is added 2627 * and returned. 2628 */ 2629 private synchronized HbckRegionInfo getOrCreateInfo(String name) { 2630 HbckRegionInfo hbi = regionInfoMap.get(name); 2631 if (hbi == null) { 2632 hbi = new HbckRegionInfo(null); 2633 regionInfoMap.put(name, hbi); 2634 } 2635 return hbi; 2636 } 2637 2638 private void checkAndFixReplication() throws ReplicationException { 2639 ReplicationChecker checker = new ReplicationChecker(getConf(), zkw, errors); 2640 checker.checkUnDeletedQueues(); 2641 2642 if (checker.hasUnDeletedQueues() && this.fixReplication) { 2643 checker.fixUnDeletedQueues(); 2644 setShouldRerun(); 2645 } 2646 } 2647 2648 /** 2649 * Check values in regionInfo for hbase:meta 2650 * Check if zero or more than one regions with hbase:meta are found. 2651 * If there are inconsistencies (i.e. zero or more than one regions 2652 * pretend to be holding the hbase:meta) try to fix that and report an error. 2653 * @throws IOException from HBaseFsckRepair functions 2654 * @throws KeeperException 2655 * @throws InterruptedException 2656 */ 2657 boolean checkMetaRegion() throws IOException, KeeperException, InterruptedException { 2658 Map<Integer, HbckRegionInfo> metaRegions = new HashMap<>(); 2659 for (HbckRegionInfo value : regionInfoMap.values()) { 2660 if (value.getMetaEntry() != null && value.getMetaEntry().isMetaRegion()) { 2661 metaRegions.put(value.getReplicaId(), value); 2662 } 2663 } 2664 int metaReplication = admin.getTableDescriptor(TableName.META_TABLE_NAME) 2665 .getRegionReplication(); 2666 boolean noProblem = true; 2667 // There will be always entries in regionInfoMap corresponding to hbase:meta & its replicas 2668 // Check the deployed servers. It should be exactly one server for each replica. 2669 for (int i = 0; i < metaReplication; i++) { 2670 HbckRegionInfo metaHbckRegionInfo = metaRegions.remove(i); 2671 List<ServerName> servers = new ArrayList<>(); 2672 if (metaHbckRegionInfo != null) { 2673 servers = metaHbckRegionInfo.getDeployedOn(); 2674 } 2675 if (servers.size() != 1) { 2676 noProblem = false; 2677 if (servers.isEmpty()) { 2678 assignMetaReplica(i); 2679 } else if (servers.size() > 1) { 2680 errors 2681 .reportError(ERROR_CODE.MULTI_META_REGION, "hbase:meta, replicaId " + 2682 metaHbckRegionInfo.getReplicaId() + " is found on more than one region."); 2683 if (shouldFixAssignments()) { 2684 errors.print("Trying to fix a problem with hbase:meta, replicaId " + 2685 metaHbckRegionInfo.getReplicaId() + ".."); 2686 setShouldRerun(); 2687 // try fix it (treat is a dupe assignment) 2688 HBaseFsckRepair 2689 .fixMultiAssignment(connection, metaHbckRegionInfo.getMetaEntry(), servers); 2690 } 2691 } 2692 } 2693 } 2694 // unassign whatever is remaining in metaRegions. They are excess replicas. 2695 for (Map.Entry<Integer, HbckRegionInfo> entry : metaRegions.entrySet()) { 2696 noProblem = false; 2697 errors.reportError(ERROR_CODE.SHOULD_NOT_BE_DEPLOYED, 2698 "hbase:meta replicas are deployed in excess. Configured " + metaReplication + 2699 ", deployed " + metaRegions.size()); 2700 if (shouldFixAssignments()) { 2701 errors.print("Trying to undeploy excess replica, replicaId: " + entry.getKey() + 2702 " of hbase:meta.."); 2703 setShouldRerun(); 2704 unassignMetaReplica(entry.getValue()); 2705 } 2706 } 2707 // if noProblem is false, rerun hbck with hopefully fixed META 2708 // if noProblem is true, no errors, so continue normally 2709 return noProblem; 2710 } 2711 2712 private void unassignMetaReplica(HbckRegionInfo hi) 2713 throws IOException, InterruptedException, KeeperException { 2714 undeployRegions(hi); 2715 ZKUtil 2716 .deleteNode(zkw, zkw.getZNodePaths().getZNodeForReplica(hi.getMetaEntry().getReplicaId())); 2717 } 2718 2719 private void assignMetaReplica(int replicaId) 2720 throws IOException, KeeperException, InterruptedException { 2721 errors.reportError(ERROR_CODE.NO_META_REGION, "hbase:meta, replicaId " + 2722 replicaId +" is not found on any region."); 2723 if (shouldFixAssignments()) { 2724 errors.print("Trying to fix a problem with hbase:meta.."); 2725 setShouldRerun(); 2726 // try to fix it (treat it as unassigned region) 2727 RegionInfo h = RegionReplicaUtil.getRegionInfoForReplica( 2728 RegionInfoBuilder.FIRST_META_REGIONINFO, replicaId); 2729 HBaseFsckRepair.fixUnassigned(admin, h); 2730 HBaseFsckRepair.waitUntilAssigned(admin, h); 2731 } 2732 } 2733 2734 /** 2735 * Scan hbase:meta, adding all regions found to the regionInfo map. 2736 * @throws IOException if an error is encountered 2737 */ 2738 boolean loadMetaEntries() throws IOException { 2739 MetaTableAccessor.Visitor visitor = new MetaTableAccessor.Visitor() { 2740 int countRecord = 1; 2741 2742 // comparator to sort KeyValues with latest modtime 2743 final Comparator<Cell> comp = new Comparator<Cell>() { 2744 @Override 2745 public int compare(Cell k1, Cell k2) { 2746 return Long.compare(k1.getTimestamp(), k2.getTimestamp()); 2747 } 2748 }; 2749 2750 @Override 2751 public boolean visit(Result result) throws IOException { 2752 try { 2753 2754 // record the latest modification of this META record 2755 long ts = Collections.max(result.listCells(), comp).getTimestamp(); 2756 RegionLocations rl = MetaTableAccessor.getRegionLocations(result); 2757 if (rl == null) { 2758 emptyRegionInfoQualifiers.add(result); 2759 errors.reportError(ERROR_CODE.EMPTY_META_CELL, 2760 "Empty REGIONINFO_QUALIFIER found in hbase:meta"); 2761 return true; 2762 } 2763 ServerName sn = null; 2764 if (rl.getRegionLocation(RegionInfo.DEFAULT_REPLICA_ID) == null || 2765 rl.getRegionLocation(RegionInfo.DEFAULT_REPLICA_ID).getRegionInfo() == null) { 2766 emptyRegionInfoQualifiers.add(result); 2767 errors.reportError(ERROR_CODE.EMPTY_META_CELL, 2768 "Empty REGIONINFO_QUALIFIER found in hbase:meta"); 2769 return true; 2770 } 2771 RegionInfo hri = rl.getRegionLocation(RegionInfo.DEFAULT_REPLICA_ID).getRegionInfo(); 2772 if (!(isTableIncluded(hri.getTable()) 2773 || hri.isMetaRegion())) { 2774 return true; 2775 } 2776 PairOfSameType<RegionInfo> daughters = MetaTableAccessor.getDaughterRegions(result); 2777 for (HRegionLocation h : rl.getRegionLocations()) { 2778 if (h == null || h.getRegionInfo() == null) { 2779 continue; 2780 } 2781 sn = h.getServerName(); 2782 hri = h.getRegionInfo(); 2783 2784 HbckRegionInfo.MetaEntry m = null; 2785 if (hri.getReplicaId() == RegionInfo.DEFAULT_REPLICA_ID) { 2786 m = new HbckRegionInfo.MetaEntry(hri, sn, ts, daughters.getFirst(), 2787 daughters.getSecond()); 2788 } else { 2789 m = new HbckRegionInfo.MetaEntry(hri, sn, ts, null, null); 2790 } 2791 HbckRegionInfo previous = regionInfoMap.get(hri.getEncodedName()); 2792 if (previous == null) { 2793 regionInfoMap.put(hri.getEncodedName(), new HbckRegionInfo(m)); 2794 } else if (previous.getMetaEntry() == null) { 2795 previous.setMetaEntry(m); 2796 } else { 2797 throw new IOException("Two entries in hbase:meta are same " + previous); 2798 } 2799 } 2800 List<RegionInfo> mergeParents = MetaTableAccessor.getMergeRegions(result.rawCells()); 2801 if (mergeParents != null) { 2802 for (RegionInfo mergeRegion : mergeParents) { 2803 if (mergeRegion != null) { 2804 // This region is already being merged 2805 HbckRegionInfo hbInfo = getOrCreateInfo(mergeRegion.getEncodedName()); 2806 hbInfo.setMerged(true); 2807 } 2808 } 2809 } 2810 2811 // show proof of progress to the user, once for every 100 records. 2812 if (countRecord % 100 == 0) { 2813 errors.progress(); 2814 } 2815 countRecord++; 2816 return true; 2817 } catch (RuntimeException e) { 2818 LOG.error("Result=" + result); 2819 throw e; 2820 } 2821 } 2822 }; 2823 if (!checkMetaOnly) { 2824 // Scan hbase:meta to pick up user regions 2825 MetaTableAccessor.fullScanRegions(connection, visitor); 2826 } 2827 2828 errors.print(""); 2829 return true; 2830 } 2831 2832 /** 2833 * Prints summary of all tables found on the system. 2834 */ 2835 private void printTableSummary(SortedMap<TableName, HbckTableInfo> tablesInfo) { 2836 StringBuilder sb = new StringBuilder(); 2837 int numOfSkippedRegions; 2838 errors.print("Summary:"); 2839 for (HbckTableInfo tInfo : tablesInfo.values()) { 2840 numOfSkippedRegions = (skippedRegions.containsKey(tInfo.getName())) ? 2841 skippedRegions.get(tInfo.getName()).size() : 0; 2842 2843 if (errors.tableHasErrors(tInfo)) { 2844 errors.print("Table " + tInfo.getName() + " is inconsistent."); 2845 } else if (numOfSkippedRegions > 0){ 2846 errors.print("Table " + tInfo.getName() + " is okay (with " 2847 + numOfSkippedRegions + " skipped regions)."); 2848 } 2849 else { 2850 errors.print("Table " + tInfo.getName() + " is okay."); 2851 } 2852 errors.print(" Number of regions: " + tInfo.getNumRegions()); 2853 if (numOfSkippedRegions > 0) { 2854 Set<String> skippedRegionStrings = skippedRegions.get(tInfo.getName()); 2855 System.out.println(" Number of skipped regions: " + numOfSkippedRegions); 2856 System.out.println(" List of skipped regions:"); 2857 for(String sr : skippedRegionStrings) { 2858 System.out.println(" " + sr); 2859 } 2860 } 2861 sb.setLength(0); // clear out existing buffer, if any. 2862 sb.append(" Deployed on: "); 2863 for (ServerName server : tInfo.deployedOn) { 2864 sb.append(" " + server.toString()); 2865 } 2866 errors.print(sb.toString()); 2867 } 2868 } 2869 2870 static HbckErrorReporter getErrorReporter(final Configuration conf) 2871 throws ClassNotFoundException { 2872 Class<? extends HbckErrorReporter> reporter = 2873 conf.getClass("hbasefsck.errorreporter", PrintingErrorReporter.class, 2874 HbckErrorReporter.class); 2875 return ReflectionUtils.newInstance(reporter, conf); 2876 } 2877 2878 static class PrintingErrorReporter implements HbckErrorReporter { 2879 public int errorCount = 0; 2880 private int showProgress; 2881 // How frequently calls to progress() will create output 2882 private static final int progressThreshold = 100; 2883 2884 Set<HbckTableInfo> errorTables = new HashSet<>(); 2885 2886 // for use by unit tests to verify which errors were discovered 2887 private ArrayList<ERROR_CODE> errorList = new ArrayList<>(); 2888 2889 @Override 2890 public void clear() { 2891 errorTables.clear(); 2892 errorList.clear(); 2893 errorCount = 0; 2894 } 2895 2896 @Override 2897 public synchronized void reportError(ERROR_CODE errorCode, String message) { 2898 if (errorCode == ERROR_CODE.WRONG_USAGE) { 2899 System.err.println(message); 2900 return; 2901 } 2902 2903 errorList.add(errorCode); 2904 if (!summary) { 2905 System.out.println("ERROR: " + message); 2906 } 2907 errorCount++; 2908 showProgress = 0; 2909 } 2910 2911 @Override 2912 public synchronized void reportError(ERROR_CODE errorCode, String message, 2913 HbckTableInfo table) { 2914 errorTables.add(table); 2915 reportError(errorCode, message); 2916 } 2917 2918 @Override 2919 public synchronized void reportError(ERROR_CODE errorCode, String message, HbckTableInfo table, 2920 HbckRegionInfo info) { 2921 errorTables.add(table); 2922 String reference = "(region " + info.getRegionNameAsString() + ")"; 2923 reportError(errorCode, reference + " " + message); 2924 } 2925 2926 @Override 2927 public synchronized void reportError(ERROR_CODE errorCode, String message, HbckTableInfo table, 2928 HbckRegionInfo info1, HbckRegionInfo info2) { 2929 errorTables.add(table); 2930 String reference = "(regions " + info1.getRegionNameAsString() 2931 + " and " + info2.getRegionNameAsString() + ")"; 2932 reportError(errorCode, reference + " " + message); 2933 } 2934 2935 @Override 2936 public synchronized void reportError(String message) { 2937 reportError(ERROR_CODE.UNKNOWN, message); 2938 } 2939 2940 /** 2941 * Report error information, but do not increment the error count. Intended for cases 2942 * where the actual error would have been reported previously. 2943 * @param message 2944 */ 2945 @Override 2946 public synchronized void report(String message) { 2947 if (! summary) { 2948 System.out.println("ERROR: " + message); 2949 } 2950 showProgress = 0; 2951 } 2952 2953 @Override 2954 public synchronized int summarize() { 2955 System.out.println(Integer.toString(errorCount) + 2956 " inconsistencies detected."); 2957 if (errorCount == 0) { 2958 System.out.println("Status: OK"); 2959 return 0; 2960 } else { 2961 System.out.println("Status: INCONSISTENT"); 2962 return -1; 2963 } 2964 } 2965 2966 @Override 2967 public ArrayList<ERROR_CODE> getErrorList() { 2968 return errorList; 2969 } 2970 2971 @Override 2972 public synchronized void print(String message) { 2973 if (!summary) { 2974 System.out.println(message); 2975 } 2976 } 2977 2978 @Override 2979 public boolean tableHasErrors(HbckTableInfo table) { 2980 return errorTables.contains(table); 2981 } 2982 2983 @Override 2984 public void resetErrors() { 2985 errorCount = 0; 2986 } 2987 2988 @Override 2989 public synchronized void detail(String message) { 2990 if (details) { 2991 System.out.println(message); 2992 } 2993 showProgress = 0; 2994 } 2995 2996 @Override 2997 public synchronized void progress() { 2998 if (showProgress++ == progressThreshold) { 2999 if (!summary) { 3000 System.out.print("."); 3001 } 3002 showProgress = 0; 3003 } 3004 } 3005 } 3006 3007 /** 3008 * Contact a region server and get all information from it 3009 */ 3010 static class WorkItemRegion implements Callable<Void> { 3011 private final HBaseFsck hbck; 3012 private final ServerName rsinfo; 3013 private final HbckErrorReporter errors; 3014 private final ClusterConnection connection; 3015 3016 WorkItemRegion(HBaseFsck hbck, ServerName info, HbckErrorReporter errors, 3017 ClusterConnection connection) { 3018 this.hbck = hbck; 3019 this.rsinfo = info; 3020 this.errors = errors; 3021 this.connection = connection; 3022 } 3023 3024 @Override 3025 public synchronized Void call() throws IOException { 3026 errors.progress(); 3027 try { 3028 BlockingInterface server = connection.getAdmin(rsinfo); 3029 3030 // list all online regions from this region server 3031 List<RegionInfo> regions = ProtobufUtil.getOnlineRegions(server); 3032 regions = filterRegions(regions); 3033 3034 if (details) { 3035 errors.detail("RegionServer: " + rsinfo.getServerName() + 3036 " number of regions: " + regions.size()); 3037 for (RegionInfo rinfo: regions) { 3038 errors.detail(" " + rinfo.getRegionNameAsString() + 3039 " id: " + rinfo.getRegionId() + 3040 " encoded_name: " + rinfo.getEncodedName() + 3041 " start: " + Bytes.toStringBinary(rinfo.getStartKey()) + 3042 " end: " + Bytes.toStringBinary(rinfo.getEndKey())); 3043 } 3044 } 3045 3046 // check to see if the existence of this region matches the region in META 3047 3048 for (RegionInfo r : regions) { 3049 HbckRegionInfo hbi = hbck.getOrCreateInfo(r.getEncodedName()); 3050 hbi.addServer(r, rsinfo); 3051 } 3052 } catch (IOException e) { // unable to connect to the region server. 3053 errors.reportError(ERROR_CODE.RS_CONNECT_FAILURE, "RegionServer: " + rsinfo.getServerName() + 3054 " Unable to fetch region information. " + e); 3055 throw e; 3056 } 3057 return null; 3058 } 3059 3060 private List<RegionInfo> filterRegions(List<RegionInfo> regions) { 3061 List<RegionInfo> ret = Lists.newArrayList(); 3062 for (RegionInfo hri : regions) { 3063 if (hri.isMetaRegion() || (!hbck.checkMetaOnly 3064 && hbck.isTableIncluded(hri.getTable()))) { 3065 ret.add(hri); 3066 } 3067 } 3068 return ret; 3069 } 3070 } 3071 3072 /** 3073 * Contact hdfs and get all information about specified table directory into 3074 * regioninfo list. 3075 */ 3076 class WorkItemHdfsDir implements Callable<Void> { 3077 private FileStatus tableDir; 3078 private HbckErrorReporter errors; 3079 private FileSystem fs; 3080 3081 WorkItemHdfsDir(FileSystem fs, HbckErrorReporter errors, FileStatus status) { 3082 this.fs = fs; 3083 this.tableDir = status; 3084 this.errors = errors; 3085 } 3086 3087 @Override 3088 public synchronized Void call() throws InterruptedException, ExecutionException { 3089 final Vector<Exception> exceptions = new Vector<>(); 3090 3091 try { 3092 final FileStatus[] regionDirs = fs.listStatus(tableDir.getPath()); 3093 final List<Future<?>> futures = new ArrayList<>(regionDirs.length); 3094 3095 for (final FileStatus regionDir : regionDirs) { 3096 errors.progress(); 3097 final String encodedName = regionDir.getPath().getName(); 3098 // ignore directories that aren't hexadecimal 3099 if (!encodedName.toLowerCase(Locale.ROOT).matches("[0-9a-f]+")) { 3100 continue; 3101 } 3102 3103 if (!exceptions.isEmpty()) { 3104 break; 3105 } 3106 3107 futures.add(executor.submit(new Runnable() { 3108 @Override 3109 public void run() { 3110 try { 3111 LOG.debug("Loading region info from hdfs:"+ regionDir.getPath()); 3112 3113 Path regioninfoFile = new Path(regionDir.getPath(), HRegionFileSystem.REGION_INFO_FILE); 3114 boolean regioninfoFileExists = fs.exists(regioninfoFile); 3115 3116 if (!regioninfoFileExists) { 3117 // As tables become larger it is more and more likely that by the time you 3118 // reach a given region that it will be gone due to region splits/merges. 3119 if (!fs.exists(regionDir.getPath())) { 3120 LOG.warn("By the time we tried to process this region dir it was already gone: " 3121 + regionDir.getPath()); 3122 return; 3123 } 3124 } 3125 3126 HbckRegionInfo hbi = HBaseFsck.this.getOrCreateInfo(encodedName); 3127 HbckRegionInfo.HdfsEntry he = new HbckRegionInfo.HdfsEntry(); 3128 synchronized (hbi) { 3129 if (hbi.getHdfsRegionDir() != null) { 3130 errors.print("Directory " + encodedName + " duplicate??" + 3131 hbi.getHdfsRegionDir()); 3132 } 3133 3134 he.regionDir = regionDir.getPath(); 3135 he.regionDirModTime = regionDir.getModificationTime(); 3136 he.hdfsRegioninfoFilePresent = regioninfoFileExists; 3137 // we add to orphan list when we attempt to read .regioninfo 3138 3139 // Set a flag if this region contains only edits 3140 // This is special case if a region is left after split 3141 he.hdfsOnlyEdits = true; 3142 FileStatus[] subDirs = fs.listStatus(regionDir.getPath()); 3143 Path ePath = WALSplitUtil.getRegionDirRecoveredEditsDir(regionDir.getPath()); 3144 for (FileStatus subDir : subDirs) { 3145 errors.progress(); 3146 String sdName = subDir.getPath().getName(); 3147 if (!sdName.startsWith(".") && !sdName.equals(ePath.getName())) { 3148 he.hdfsOnlyEdits = false; 3149 break; 3150 } 3151 } 3152 hbi.setHdfsEntry(he); 3153 } 3154 } catch (Exception e) { 3155 LOG.error("Could not load region dir", e); 3156 exceptions.add(e); 3157 } 3158 } 3159 })); 3160 } 3161 3162 // Ensure all pending tasks are complete (or that we run into an exception) 3163 for (Future<?> f : futures) { 3164 if (!exceptions.isEmpty()) { 3165 break; 3166 } 3167 try { 3168 f.get(); 3169 } catch (ExecutionException e) { 3170 LOG.error("Unexpected exec exception! Should've been caught already. (Bug?)", e); 3171 // Shouldn't happen, we already logged/caught any exceptions in the Runnable 3172 }; 3173 } 3174 } catch (IOException e) { 3175 LOG.error("Cannot execute WorkItemHdfsDir for " + tableDir, e); 3176 exceptions.add(e); 3177 } finally { 3178 if (!exceptions.isEmpty()) { 3179 errors.reportError(ERROR_CODE.RS_CONNECT_FAILURE, "Table Directory: " 3180 + tableDir.getPath().getName() 3181 + " Unable to fetch all HDFS region information. "); 3182 // Just throw the first exception as an indication something bad happened 3183 // Don't need to propagate all the exceptions, we already logged them all anyway 3184 throw new ExecutionException("First exception in WorkItemHdfsDir", exceptions.firstElement()); 3185 } 3186 } 3187 return null; 3188 } 3189 } 3190 3191 /** 3192 * Contact hdfs and get all information about specified table directory into 3193 * regioninfo list. 3194 */ 3195 static class WorkItemHdfsRegionInfo implements Callable<Void> { 3196 private HbckRegionInfo hbi; 3197 private HBaseFsck hbck; 3198 private HbckErrorReporter errors; 3199 3200 WorkItemHdfsRegionInfo(HbckRegionInfo hbi, HBaseFsck hbck, HbckErrorReporter errors) { 3201 this.hbi = hbi; 3202 this.hbck = hbck; 3203 this.errors = errors; 3204 } 3205 3206 @Override 3207 public synchronized Void call() throws IOException { 3208 // only load entries that haven't been loaded yet. 3209 if (hbi.getHdfsHRI() == null) { 3210 try { 3211 errors.progress(); 3212 hbi.loadHdfsRegioninfo(hbck.getConf()); 3213 } catch (IOException ioe) { 3214 String msg = "Orphan region in HDFS: Unable to load .regioninfo from table " 3215 + hbi.getTableName() + " in hdfs dir " 3216 + hbi.getHdfsRegionDir() 3217 + "! It may be an invalid format or version file. Treating as " 3218 + "an orphaned regiondir."; 3219 errors.reportError(ERROR_CODE.ORPHAN_HDFS_REGION, msg); 3220 try { 3221 hbck.debugLsr(hbi.getHdfsRegionDir()); 3222 } catch (IOException ioe2) { 3223 LOG.error("Unable to read directory " + hbi.getHdfsRegionDir(), ioe2); 3224 throw ioe2; 3225 } 3226 hbck.orphanHdfsDirs.add(hbi); 3227 throw ioe; 3228 } 3229 } 3230 return null; 3231 } 3232 }; 3233 3234 /** 3235 * Display the full report from fsck. This displays all live and dead region 3236 * servers, and all known regions. 3237 */ 3238 public static void setDisplayFullReport() { 3239 details = true; 3240 } 3241 3242 public static boolean shouldDisplayFullReport() { 3243 return details; 3244 } 3245 3246 /** 3247 * Set exclusive mode. 3248 */ 3249 public static void setForceExclusive() { 3250 forceExclusive = true; 3251 } 3252 3253 /** 3254 * Only one instance of hbck can modify HBase at a time. 3255 */ 3256 public boolean isExclusive() { 3257 return fixAny || forceExclusive; 3258 } 3259 3260 /** 3261 * Set summary mode. 3262 * Print only summary of the tables and status (OK or INCONSISTENT) 3263 */ 3264 static void setSummary() { 3265 summary = true; 3266 } 3267 3268 /** 3269 * Set hbase:meta check mode. 3270 * Print only info about hbase:meta table deployment/state 3271 */ 3272 void setCheckMetaOnly() { 3273 checkMetaOnly = true; 3274 } 3275 3276 /** 3277 * Set region boundaries check mode. 3278 */ 3279 void setRegionBoundariesCheck() { 3280 checkRegionBoundaries = true; 3281 } 3282 3283 /** 3284 * Set replication fix mode. 3285 */ 3286 public void setFixReplication(boolean shouldFix) { 3287 fixReplication = shouldFix; 3288 fixAny |= shouldFix; 3289 } 3290 3291 public void setCleanReplicationBarrier(boolean shouldClean) { 3292 cleanReplicationBarrier = shouldClean; 3293 } 3294 3295 /** 3296 * Check if we should rerun fsck again. This checks if we've tried to 3297 * fix something and we should rerun fsck tool again. 3298 * Display the full report from fsck. This displays all live and dead 3299 * region servers, and all known regions. 3300 */ 3301 void setShouldRerun() { 3302 rerun = true; 3303 } 3304 3305 public boolean shouldRerun() { 3306 return rerun; 3307 } 3308 3309 /** 3310 * Fix inconsistencies found by fsck. This should try to fix errors (if any) 3311 * found by fsck utility. 3312 */ 3313 public void setFixAssignments(boolean shouldFix) { 3314 fixAssignments = shouldFix; 3315 fixAny |= shouldFix; 3316 } 3317 3318 boolean shouldFixAssignments() { 3319 return fixAssignments; 3320 } 3321 3322 public void setFixMeta(boolean shouldFix) { 3323 fixMeta = shouldFix; 3324 fixAny |= shouldFix; 3325 } 3326 3327 boolean shouldFixMeta() { 3328 return fixMeta; 3329 } 3330 3331 public void setFixEmptyMetaCells(boolean shouldFix) { 3332 fixEmptyMetaCells = shouldFix; 3333 fixAny |= shouldFix; 3334 } 3335 3336 boolean shouldFixEmptyMetaCells() { 3337 return fixEmptyMetaCells; 3338 } 3339 3340 public void setCheckHdfs(boolean checking) { 3341 checkHdfs = checking; 3342 } 3343 3344 boolean shouldCheckHdfs() { 3345 return checkHdfs; 3346 } 3347 3348 public void setFixHdfsHoles(boolean shouldFix) { 3349 fixHdfsHoles = shouldFix; 3350 fixAny |= shouldFix; 3351 } 3352 3353 boolean shouldFixHdfsHoles() { 3354 return fixHdfsHoles; 3355 } 3356 3357 public void setFixTableOrphans(boolean shouldFix) { 3358 fixTableOrphans = shouldFix; 3359 fixAny |= shouldFix; 3360 } 3361 3362 boolean shouldFixTableOrphans() { 3363 return fixTableOrphans; 3364 } 3365 3366 public void setFixHdfsOverlaps(boolean shouldFix) { 3367 fixHdfsOverlaps = shouldFix; 3368 fixAny |= shouldFix; 3369 } 3370 3371 boolean shouldFixHdfsOverlaps() { 3372 return fixHdfsOverlaps; 3373 } 3374 3375 public void setFixHdfsOrphans(boolean shouldFix) { 3376 fixHdfsOrphans = shouldFix; 3377 fixAny |= shouldFix; 3378 } 3379 3380 boolean shouldFixHdfsOrphans() { 3381 return fixHdfsOrphans; 3382 } 3383 3384 public void setFixVersionFile(boolean shouldFix) { 3385 fixVersionFile = shouldFix; 3386 fixAny |= shouldFix; 3387 } 3388 3389 public boolean shouldFixVersionFile() { 3390 return fixVersionFile; 3391 } 3392 3393 public void setSidelineBigOverlaps(boolean sbo) { 3394 this.sidelineBigOverlaps = sbo; 3395 } 3396 3397 public boolean shouldSidelineBigOverlaps() { 3398 return sidelineBigOverlaps; 3399 } 3400 3401 public void setFixSplitParents(boolean shouldFix) { 3402 fixSplitParents = shouldFix; 3403 fixAny |= shouldFix; 3404 } 3405 3406 public void setRemoveParents(boolean shouldFix) { 3407 removeParents = shouldFix; 3408 fixAny |= shouldFix; 3409 } 3410 3411 boolean shouldFixSplitParents() { 3412 return fixSplitParents; 3413 } 3414 3415 boolean shouldRemoveParents() { 3416 return removeParents; 3417 } 3418 3419 public void setFixReferenceFiles(boolean shouldFix) { 3420 fixReferenceFiles = shouldFix; 3421 fixAny |= shouldFix; 3422 } 3423 3424 boolean shouldFixReferenceFiles() { 3425 return fixReferenceFiles; 3426 } 3427 3428 public void setFixHFileLinks(boolean shouldFix) { 3429 fixHFileLinks = shouldFix; 3430 fixAny |= shouldFix; 3431 } 3432 3433 boolean shouldFixHFileLinks() { 3434 return fixHFileLinks; 3435 } 3436 3437 public boolean shouldIgnorePreCheckPermission() { 3438 return !fixAny || ignorePreCheckPermission; 3439 } 3440 3441 public void setIgnorePreCheckPermission(boolean ignorePreCheckPermission) { 3442 this.ignorePreCheckPermission = ignorePreCheckPermission; 3443 } 3444 3445 /** 3446 * @param mm maximum number of regions to merge into a single region. 3447 */ 3448 public void setMaxMerge(int mm) { 3449 this.maxMerge = mm; 3450 } 3451 3452 public int getMaxMerge() { 3453 return maxMerge; 3454 } 3455 3456 public void setMaxOverlapsToSideline(int mo) { 3457 this.maxOverlapsToSideline = mo; 3458 } 3459 3460 public int getMaxOverlapsToSideline() { 3461 return maxOverlapsToSideline; 3462 } 3463 3464 /** 3465 * Only check/fix tables specified by the list, 3466 * Empty list means all tables are included. 3467 */ 3468 boolean isTableIncluded(TableName table) { 3469 return (tablesIncluded.isEmpty()) || tablesIncluded.contains(table); 3470 } 3471 3472 public void includeTable(TableName table) { 3473 tablesIncluded.add(table); 3474 } 3475 3476 Set<TableName> getIncludedTables() { 3477 return new HashSet<>(tablesIncluded); 3478 } 3479 3480 /** 3481 * We are interested in only those tables that have not changed their state in 3482 * hbase:meta during the last few seconds specified by hbase.admin.fsck.timelag 3483 * @param seconds - the time in seconds 3484 */ 3485 public void setTimeLag(long seconds) { 3486 timelag = seconds * 1000; // convert to milliseconds 3487 } 3488 3489 /** 3490 * 3491 * @param sidelineDir - HDFS path to sideline data 3492 */ 3493 public void setSidelineDir(String sidelineDir) { 3494 this.sidelineDir = new Path(sidelineDir); 3495 } 3496 3497 protected HFileCorruptionChecker createHFileCorruptionChecker(boolean sidelineCorruptHFiles) throws IOException { 3498 return new HFileCorruptionChecker(getConf(), executor, sidelineCorruptHFiles); 3499 } 3500 3501 public HFileCorruptionChecker getHFilecorruptionChecker() { 3502 return hfcc; 3503 } 3504 3505 public void setHFileCorruptionChecker(HFileCorruptionChecker hfcc) { 3506 this.hfcc = hfcc; 3507 } 3508 3509 public void setRetCode(int code) { 3510 this.retcode = code; 3511 } 3512 3513 public int getRetCode() { 3514 return retcode; 3515 } 3516 3517 protected HBaseFsck printUsageAndExit() { 3518 StringWriter sw = new StringWriter(2048); 3519 PrintWriter out = new PrintWriter(sw); 3520 out.println(""); 3521 out.println("-----------------------------------------------------------------------"); 3522 out.println("NOTE: As of HBase version 2.0, the hbck tool is significantly changed."); 3523 out.println("In general, all Read-Only options are supported and can be be used"); 3524 out.println("safely. Most -fix/ -repair options are NOT supported. Please see usage"); 3525 out.println("below for details on which options are not supported."); 3526 out.println("-----------------------------------------------------------------------"); 3527 out.println(""); 3528 out.println("Usage: fsck [opts] {only tables}"); 3529 out.println(" where [opts] are:"); 3530 out.println(" -help Display help options (this)"); 3531 out.println(" -details Display full report of all regions."); 3532 out.println(" -timelag <timeInSeconds> Process only regions that " + 3533 " have not experienced any metadata updates in the last " + 3534 " <timeInSeconds> seconds."); 3535 out.println(" -sleepBeforeRerun <timeInSeconds> Sleep this many seconds" + 3536 " before checking if the fix worked if run with -fix"); 3537 out.println(" -summary Print only summary of the tables and status."); 3538 out.println(" -metaonly Only check the state of the hbase:meta table."); 3539 out.println(" -sidelineDir <hdfs://> HDFS path to backup existing meta."); 3540 out.println(" -boundaries Verify that regions boundaries are the same between META and store files."); 3541 out.println(" -exclusive Abort if another hbck is exclusive or fixing."); 3542 3543 out.println(""); 3544 out.println(" Datafile Repair options: (expert features, use with caution!)"); 3545 out.println(" -checkCorruptHFiles Check all Hfiles by opening them to make sure they are valid"); 3546 out.println(" -sidelineCorruptHFiles Quarantine corrupted HFiles. implies -checkCorruptHFiles"); 3547 3548 out.println(""); 3549 out.println(" Replication options"); 3550 out.println(" -fixReplication Deletes replication queues for removed peers"); 3551 3552 out.println(""); 3553 out.println(" Metadata Repair options supported as of version 2.0: (expert features, use with caution!)"); 3554 out.println(" -fixVersionFile Try to fix missing hbase.version file in hdfs."); 3555 out.println(" -fixReferenceFiles Try to offline lingering reference store files"); 3556 out.println(" -fixHFileLinks Try to offline lingering HFileLinks"); 3557 out.println(" -noHdfsChecking Don't load/check region info from HDFS." 3558 + " Assumes hbase:meta region info is good. Won't check/fix any HDFS issue, e.g. hole, orphan, or overlap"); 3559 out.println(" -ignorePreCheckPermission ignore filesystem permission pre-check"); 3560 3561 out.println(""); 3562 out.println("NOTE: Following options are NOT supported as of HBase version 2.0+."); 3563 out.println(""); 3564 out.println(" UNSUPPORTED Metadata Repair options: (expert features, use with caution!)"); 3565 out.println(" -fix Try to fix region assignments. This is for backwards compatiblity"); 3566 out.println(" -fixAssignments Try to fix region assignments. Replaces the old -fix"); 3567 out.println(" -fixMeta Try to fix meta problems. This assumes HDFS region info is good."); 3568 out.println(" -fixHdfsHoles Try to fix region holes in hdfs."); 3569 out.println(" -fixHdfsOrphans Try to fix region dirs with no .regioninfo file in hdfs"); 3570 out.println(" -fixTableOrphans Try to fix table dirs with no .tableinfo file in hdfs (online mode only)"); 3571 out.println(" -fixHdfsOverlaps Try to fix region overlaps in hdfs."); 3572 out.println(" -maxMerge <n> When fixing region overlaps, allow at most <n> regions to merge. (n=" + DEFAULT_MAX_MERGE +" by default)"); 3573 out.println(" -sidelineBigOverlaps When fixing region overlaps, allow to sideline big overlaps"); 3574 out.println(" -maxOverlapsToSideline <n> When fixing region overlaps, allow at most <n> regions to sideline per group. (n=" + DEFAULT_OVERLAPS_TO_SIDELINE +" by default)"); 3575 out.println(" -fixSplitParents Try to force offline split parents to be online."); 3576 out.println(" -removeParents Try to offline and sideline lingering parents and keep daughter regions."); 3577 out.println(" -fixEmptyMetaCells Try to fix hbase:meta entries not referencing any region" 3578 + " (empty REGIONINFO_QUALIFIER rows)"); 3579 3580 out.println(""); 3581 out.println(" UNSUPPORTED Metadata Repair shortcuts"); 3582 out.println(" -repair Shortcut for -fixAssignments -fixMeta -fixHdfsHoles " + 3583 "-fixHdfsOrphans -fixHdfsOverlaps -fixVersionFile -sidelineBigOverlaps -fixReferenceFiles" + 3584 "-fixHFileLinks"); 3585 out.println(" -repairHoles Shortcut for -fixAssignments -fixMeta -fixHdfsHoles"); 3586 out.println(""); 3587 out.println(" Replication options"); 3588 out.println(" -fixReplication Deletes replication queues for removed peers"); 3589 out.println(" -cleanReplicationBrarier [tableName] clean the replication barriers " + 3590 "of a specified table, tableName is required"); 3591 out.flush(); 3592 errors.reportError(ERROR_CODE.WRONG_USAGE, sw.toString()); 3593 3594 setRetCode(-2); 3595 return this; 3596 } 3597 3598 /** 3599 * Main program 3600 * 3601 * @param args 3602 * @throws Exception 3603 */ 3604 public static void main(String[] args) throws Exception { 3605 // create a fsck object 3606 Configuration conf = HBaseConfiguration.create(); 3607 Path hbasedir = CommonFSUtils.getRootDir(conf); 3608 URI defaultFs = hbasedir.getFileSystem(conf).getUri(); 3609 CommonFSUtils.setFsDefault(conf, new Path(defaultFs)); 3610 int ret = ToolRunner.run(new HBaseFsckTool(conf), args); 3611 System.exit(ret); 3612 } 3613 3614 /** 3615 * This is a Tool wrapper that gathers -Dxxx=yyy configuration settings from the command line. 3616 */ 3617 static class HBaseFsckTool extends Configured implements Tool { 3618 HBaseFsckTool(Configuration conf) { super(conf); } 3619 @Override 3620 public int run(String[] args) throws Exception { 3621 HBaseFsck hbck = new HBaseFsck(getConf()); 3622 hbck.exec(hbck.executor, args); 3623 hbck.close(); 3624 return hbck.getRetCode(); 3625 } 3626 }; 3627 3628 3629 public HBaseFsck exec(ExecutorService exec, String[] args) 3630 throws KeeperException, IOException, InterruptedException, ReplicationException { 3631 long sleepBeforeRerun = DEFAULT_SLEEP_BEFORE_RERUN; 3632 3633 boolean checkCorruptHFiles = false; 3634 boolean sidelineCorruptHFiles = false; 3635 3636 // Process command-line args. 3637 for (int i = 0; i < args.length; i++) { 3638 String cmd = args[i]; 3639 if (cmd.equals("-help") || cmd.equals("-h")) { 3640 return printUsageAndExit(); 3641 } else if (cmd.equals("-details")) { 3642 setDisplayFullReport(); 3643 } else if (cmd.equals("-exclusive")) { 3644 setForceExclusive(); 3645 } else if (cmd.equals("-timelag")) { 3646 if (i == args.length - 1) { 3647 errors.reportError(ERROR_CODE.WRONG_USAGE, "HBaseFsck: -timelag needs a value."); 3648 return printUsageAndExit(); 3649 } 3650 try { 3651 long timelag = Long.parseLong(args[++i]); 3652 setTimeLag(timelag); 3653 } catch (NumberFormatException e) { 3654 errors.reportError(ERROR_CODE.WRONG_USAGE, "-timelag needs a numeric value."); 3655 return printUsageAndExit(); 3656 } 3657 } else if (cmd.equals("-sleepBeforeRerun")) { 3658 if (i == args.length - 1) { 3659 errors.reportError(ERROR_CODE.WRONG_USAGE, 3660 "HBaseFsck: -sleepBeforeRerun needs a value."); 3661 return printUsageAndExit(); 3662 } 3663 try { 3664 sleepBeforeRerun = Long.parseLong(args[++i]); 3665 } catch (NumberFormatException e) { 3666 errors.reportError(ERROR_CODE.WRONG_USAGE, "-sleepBeforeRerun needs a numeric value."); 3667 return printUsageAndExit(); 3668 } 3669 } else if (cmd.equals("-sidelineDir")) { 3670 if (i == args.length - 1) { 3671 errors.reportError(ERROR_CODE.WRONG_USAGE, "HBaseFsck: -sidelineDir needs a value."); 3672 return printUsageAndExit(); 3673 } 3674 setSidelineDir(args[++i]); 3675 } else if (cmd.equals("-fix")) { 3676 errors.reportError(ERROR_CODE.WRONG_USAGE, 3677 "This option is deprecated, please use -fixAssignments instead."); 3678 setFixAssignments(true); 3679 } else if (cmd.equals("-fixAssignments")) { 3680 setFixAssignments(true); 3681 } else if (cmd.equals("-fixMeta")) { 3682 setFixMeta(true); 3683 } else if (cmd.equals("-noHdfsChecking")) { 3684 setCheckHdfs(false); 3685 } else if (cmd.equals("-fixHdfsHoles")) { 3686 setFixHdfsHoles(true); 3687 } else if (cmd.equals("-fixHdfsOrphans")) { 3688 setFixHdfsOrphans(true); 3689 } else if (cmd.equals("-fixTableOrphans")) { 3690 setFixTableOrphans(true); 3691 } else if (cmd.equals("-fixHdfsOverlaps")) { 3692 setFixHdfsOverlaps(true); 3693 } else if (cmd.equals("-fixVersionFile")) { 3694 setFixVersionFile(true); 3695 } else if (cmd.equals("-sidelineBigOverlaps")) { 3696 setSidelineBigOverlaps(true); 3697 } else if (cmd.equals("-fixSplitParents")) { 3698 setFixSplitParents(true); 3699 } else if (cmd.equals("-removeParents")) { 3700 setRemoveParents(true); 3701 } else if (cmd.equals("-ignorePreCheckPermission")) { 3702 setIgnorePreCheckPermission(true); 3703 } else if (cmd.equals("-checkCorruptHFiles")) { 3704 checkCorruptHFiles = true; 3705 } else if (cmd.equals("-sidelineCorruptHFiles")) { 3706 sidelineCorruptHFiles = true; 3707 } else if (cmd.equals("-fixReferenceFiles")) { 3708 setFixReferenceFiles(true); 3709 } else if (cmd.equals("-fixHFileLinks")) { 3710 setFixHFileLinks(true); 3711 } else if (cmd.equals("-fixEmptyMetaCells")) { 3712 setFixEmptyMetaCells(true); 3713 } else if (cmd.equals("-repair")) { 3714 // this attempts to merge overlapping hdfs regions, needs testing 3715 // under load 3716 setFixHdfsHoles(true); 3717 setFixHdfsOrphans(true); 3718 setFixMeta(true); 3719 setFixAssignments(true); 3720 setFixHdfsOverlaps(true); 3721 setFixVersionFile(true); 3722 setSidelineBigOverlaps(true); 3723 setFixSplitParents(false); 3724 setCheckHdfs(true); 3725 setFixReferenceFiles(true); 3726 setFixHFileLinks(true); 3727 } else if (cmd.equals("-repairHoles")) { 3728 // this will make all missing hdfs regions available but may lose data 3729 setFixHdfsHoles(true); 3730 setFixHdfsOrphans(false); 3731 setFixMeta(true); 3732 setFixAssignments(true); 3733 setFixHdfsOverlaps(false); 3734 setSidelineBigOverlaps(false); 3735 setFixSplitParents(false); 3736 setCheckHdfs(true); 3737 } else if (cmd.equals("-maxOverlapsToSideline")) { 3738 if (i == args.length - 1) { 3739 errors.reportError(ERROR_CODE.WRONG_USAGE, 3740 "-maxOverlapsToSideline needs a numeric value argument."); 3741 return printUsageAndExit(); 3742 } 3743 try { 3744 int maxOverlapsToSideline = Integer.parseInt(args[++i]); 3745 setMaxOverlapsToSideline(maxOverlapsToSideline); 3746 } catch (NumberFormatException e) { 3747 errors.reportError(ERROR_CODE.WRONG_USAGE, 3748 "-maxOverlapsToSideline needs a numeric value argument."); 3749 return printUsageAndExit(); 3750 } 3751 } else if (cmd.equals("-maxMerge")) { 3752 if (i == args.length - 1) { 3753 errors.reportError(ERROR_CODE.WRONG_USAGE, 3754 "-maxMerge needs a numeric value argument."); 3755 return printUsageAndExit(); 3756 } 3757 try { 3758 int maxMerge = Integer.parseInt(args[++i]); 3759 setMaxMerge(maxMerge); 3760 } catch (NumberFormatException e) { 3761 errors.reportError(ERROR_CODE.WRONG_USAGE, 3762 "-maxMerge needs a numeric value argument."); 3763 return printUsageAndExit(); 3764 } 3765 } else if (cmd.equals("-summary")) { 3766 setSummary(); 3767 } else if (cmd.equals("-metaonly")) { 3768 setCheckMetaOnly(); 3769 } else if (cmd.equals("-boundaries")) { 3770 setRegionBoundariesCheck(); 3771 } else if (cmd.equals("-fixReplication")) { 3772 setFixReplication(true); 3773 } else if (cmd.equals("-cleanReplicationBarrier")) { 3774 setCleanReplicationBarrier(true); 3775 if(args[++i].startsWith("-")){ 3776 printUsageAndExit(); 3777 } 3778 setCleanReplicationBarrierTable(args[i]); 3779 } else if (cmd.startsWith("-")) { 3780 errors.reportError(ERROR_CODE.WRONG_USAGE, "Unrecognized option:" + cmd); 3781 return printUsageAndExit(); 3782 } else { 3783 includeTable(TableName.valueOf(cmd)); 3784 errors.print("Allow checking/fixes for table: " + cmd); 3785 } 3786 } 3787 3788 errors.print("HBaseFsck command line options: " + StringUtils.join(args, " ")); 3789 3790 // pre-check current user has FS write permission or not 3791 try { 3792 preCheckPermission(); 3793 } catch (IOException ioe) { 3794 Runtime.getRuntime().exit(-1); 3795 } 3796 3797 // do the real work of hbck 3798 connect(); 3799 3800 // after connecting to server above, we have server version 3801 // check if unsupported option is specified based on server version 3802 if (!isOptionsSupported(args)) { 3803 return printUsageAndExit(); 3804 } 3805 3806 try { 3807 // if corrupt file mode is on, first fix them since they may be opened later 3808 if (checkCorruptHFiles || sidelineCorruptHFiles) { 3809 LOG.info("Checking all hfiles for corruption"); 3810 HFileCorruptionChecker hfcc = createHFileCorruptionChecker(sidelineCorruptHFiles); 3811 setHFileCorruptionChecker(hfcc); // so we can get result 3812 Collection<TableName> tables = getIncludedTables(); 3813 Collection<Path> tableDirs = new ArrayList<>(); 3814 Path rootdir = CommonFSUtils.getRootDir(getConf()); 3815 if (tables.size() > 0) { 3816 for (TableName t : tables) { 3817 tableDirs.add(CommonFSUtils.getTableDir(rootdir, t)); 3818 } 3819 } else { 3820 tableDirs = FSUtils.getTableDirs(CommonFSUtils.getCurrentFileSystem(getConf()), rootdir); 3821 } 3822 hfcc.checkTables(tableDirs); 3823 hfcc.report(errors); 3824 } 3825 3826 // check and fix table integrity, region consistency. 3827 int code = onlineHbck(); 3828 setRetCode(code); 3829 // If we have changed the HBase state it is better to run hbck again 3830 // to see if we haven't broken something else in the process. 3831 // We run it only once more because otherwise we can easily fall into 3832 // an infinite loop. 3833 if (shouldRerun()) { 3834 try { 3835 LOG.info("Sleeping " + sleepBeforeRerun + "ms before re-checking after fix..."); 3836 Thread.sleep(sleepBeforeRerun); 3837 } catch (InterruptedException ie) { 3838 LOG.warn("Interrupted while sleeping"); 3839 return this; 3840 } 3841 // Just report 3842 setFixAssignments(false); 3843 setFixMeta(false); 3844 setFixHdfsHoles(false); 3845 setFixHdfsOverlaps(false); 3846 setFixVersionFile(false); 3847 setFixTableOrphans(false); 3848 errors.resetErrors(); 3849 code = onlineHbck(); 3850 setRetCode(code); 3851 } 3852 } finally { 3853 IOUtils.closeQuietly(this); 3854 } 3855 return this; 3856 } 3857 3858 private boolean isOptionsSupported(String[] args) { 3859 boolean result = true; 3860 String hbaseServerVersion = status.getHBaseVersion(); 3861 if (VersionInfo.compareVersion("2.any.any", hbaseServerVersion) < 0) { 3862 // Process command-line args. 3863 for (String arg : args) { 3864 if (unsupportedOptionsInV2.contains(arg)) { 3865 errors.reportError(ERROR_CODE.UNSUPPORTED_OPTION, 3866 "option '" + arg + "' is not " + "supportted!"); 3867 result = false; 3868 break; 3869 } 3870 } 3871 } 3872 return result; 3873 } 3874 3875 public void setCleanReplicationBarrierTable(String cleanReplicationBarrierTable) { 3876 this.cleanReplicationBarrierTable = TableName.valueOf(cleanReplicationBarrierTable); 3877 } 3878 3879 public void cleanReplicationBarrier() throws IOException { 3880 if (!cleanReplicationBarrier || cleanReplicationBarrierTable == null) { 3881 return; 3882 } 3883 if (cleanReplicationBarrierTable.isSystemTable()) { 3884 errors.reportError(ERROR_CODE.INVALID_TABLE, 3885 "invalid table: " + cleanReplicationBarrierTable); 3886 return; 3887 } 3888 3889 boolean isGlobalScope = false; 3890 try { 3891 isGlobalScope = admin.getDescriptor(cleanReplicationBarrierTable).hasGlobalReplicationScope(); 3892 } catch (TableNotFoundException e) { 3893 LOG.info("we may need to clean some erroneous data due to bugs"); 3894 } 3895 3896 if (isGlobalScope) { 3897 errors.reportError(ERROR_CODE.INVALID_TABLE, 3898 "table's replication scope is global: " + cleanReplicationBarrierTable); 3899 return; 3900 } 3901 List<byte[]> regionNames = new ArrayList<>(); 3902 Scan barrierScan = new Scan(); 3903 barrierScan.setCaching(100); 3904 barrierScan.addFamily(HConstants.REPLICATION_BARRIER_FAMILY); 3905 barrierScan 3906 .withStartRow(MetaTableAccessor.getTableStartRowForMeta(cleanReplicationBarrierTable, 3907 MetaTableAccessor.QueryType.REGION)) 3908 .withStopRow(MetaTableAccessor.getTableStopRowForMeta(cleanReplicationBarrierTable, 3909 MetaTableAccessor.QueryType.REGION)); 3910 Result result; 3911 try (ResultScanner scanner = meta.getScanner(barrierScan)) { 3912 while ((result = scanner.next()) != null) { 3913 regionNames.add(result.getRow()); 3914 } 3915 } 3916 if (regionNames.size() <= 0) { 3917 errors.reportError(ERROR_CODE.INVALID_TABLE, 3918 "there is no barriers of this table: " + cleanReplicationBarrierTable); 3919 return; 3920 } 3921 ReplicationQueueStorage queueStorage = 3922 ReplicationStorageFactory.getReplicationQueueStorage(zkw, getConf()); 3923 List<ReplicationPeerDescription> peerDescriptions = admin.listReplicationPeers(); 3924 if (peerDescriptions != null && peerDescriptions.size() > 0) { 3925 List<String> peers = peerDescriptions.stream() 3926 .filter(peerConfig -> peerConfig.getPeerConfig() 3927 .needToReplicate(cleanReplicationBarrierTable)) 3928 .map(peerConfig -> peerConfig.getPeerId()).collect(Collectors.toList()); 3929 try { 3930 List<String> batch = new ArrayList<>(); 3931 for (String peer : peers) { 3932 for (byte[] regionName : regionNames) { 3933 batch.add(RegionInfo.encodeRegionName(regionName)); 3934 if (batch.size() % 100 == 0) { 3935 queueStorage.removeLastSequenceIds(peer, batch); 3936 batch.clear(); 3937 } 3938 } 3939 if (batch.size() > 0) { 3940 queueStorage.removeLastSequenceIds(peer, batch); 3941 batch.clear(); 3942 } 3943 } 3944 } catch (ReplicationException re) { 3945 throw new IOException(re); 3946 } 3947 } 3948 for (byte[] regionName : regionNames) { 3949 meta.delete(new Delete(regionName).addFamily(HConstants.REPLICATION_BARRIER_FAMILY)); 3950 } 3951 setShouldRerun(); 3952 } 3953 3954 /** 3955 * ls -r for debugging purposes 3956 */ 3957 void debugLsr(Path p) throws IOException { 3958 debugLsr(getConf(), p, errors); 3959 } 3960 3961 /** 3962 * ls -r for debugging purposes 3963 */ 3964 public static void debugLsr(Configuration conf, 3965 Path p) throws IOException { 3966 debugLsr(conf, p, new PrintingErrorReporter()); 3967 } 3968 3969 /** 3970 * ls -r for debugging purposes 3971 */ 3972 public static void debugLsr(Configuration conf, 3973 Path p, HbckErrorReporter errors) throws IOException { 3974 if (!LOG.isDebugEnabled() || p == null) { 3975 return; 3976 } 3977 FileSystem fs = p.getFileSystem(conf); 3978 3979 if (!fs.exists(p)) { 3980 // nothing 3981 return; 3982 } 3983 errors.print(p.toString()); 3984 3985 if (fs.isFile(p)) { 3986 return; 3987 } 3988 3989 if (fs.getFileStatus(p).isDirectory()) { 3990 FileStatus[] fss= fs.listStatus(p); 3991 for (FileStatus status : fss) { 3992 debugLsr(conf, status.getPath(), errors); 3993 } 3994 } 3995 } 3996}