2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org
.apache
.hadoop
.hbase
.util
;
20 import java
.io
.Closeable
;
21 import java
.io
.FileNotFoundException
;
22 import java
.io
.IOException
;
23 import java
.io
.InterruptedIOException
;
24 import java
.io
.PrintWriter
;
25 import java
.io
.StringWriter
;
26 import java
.net
.InetAddress
;
28 import java
.util
.ArrayList
;
29 import java
.util
.Collection
;
30 import java
.util
.Collections
;
31 import java
.util
.Comparator
;
32 import java
.util
.EnumSet
;
33 import java
.util
.HashMap
;
34 import java
.util
.HashSet
;
35 import java
.util
.Iterator
;
36 import java
.util
.List
;
37 import java
.util
.Locale
;
39 import java
.util
.Map
.Entry
;
40 import java
.util
.Objects
;
41 import java
.util
.Optional
;
43 import java
.util
.SortedMap
;
44 import java
.util
.TreeMap
;
45 import java
.util
.Vector
;
46 import java
.util
.concurrent
.Callable
;
47 import java
.util
.concurrent
.ConcurrentSkipListMap
;
48 import java
.util
.concurrent
.ExecutionException
;
49 import java
.util
.concurrent
.ExecutorService
;
50 import java
.util
.concurrent
.Executors
;
51 import java
.util
.concurrent
.Future
;
52 import java
.util
.concurrent
.FutureTask
;
53 import java
.util
.concurrent
.ScheduledThreadPoolExecutor
;
54 import java
.util
.concurrent
.TimeUnit
;
55 import java
.util
.concurrent
.TimeoutException
;
56 import java
.util
.concurrent
.atomic
.AtomicBoolean
;
57 import java
.util
.concurrent
.atomic
.AtomicInteger
;
58 import java
.util
.stream
.Collectors
;
59 import org
.apache
.commons
.io
.IOUtils
;
60 import org
.apache
.commons
.lang3
.StringUtils
;
61 import org
.apache
.hadoop
.conf
.Configuration
;
62 import org
.apache
.hadoop
.conf
.Configured
;
63 import org
.apache
.hadoop
.fs
.FSDataOutputStream
;
64 import org
.apache
.hadoop
.fs
.FileStatus
;
65 import org
.apache
.hadoop
.fs
.FileSystem
;
66 import org
.apache
.hadoop
.fs
.Path
;
67 import org
.apache
.hadoop
.fs
.permission
.FsAction
;
68 import org
.apache
.hadoop
.fs
.permission
.FsPermission
;
69 import org
.apache
.hadoop
.hbase
.Abortable
;
70 import org
.apache
.hadoop
.hbase
.CatalogFamilyFormat
;
71 import org
.apache
.hadoop
.hbase
.Cell
;
72 import org
.apache
.hadoop
.hbase
.CellUtil
;
73 import org
.apache
.hadoop
.hbase
.ClientMetaTableAccessor
;
74 import org
.apache
.hadoop
.hbase
.ClusterMetrics
;
75 import org
.apache
.hadoop
.hbase
.ClusterMetrics
.Option
;
76 import org
.apache
.hadoop
.hbase
.HBaseConfiguration
;
77 import org
.apache
.hadoop
.hbase
.HBaseInterfaceAudience
;
78 import org
.apache
.hadoop
.hbase
.HConstants
;
79 import org
.apache
.hadoop
.hbase
.HRegionLocation
;
80 import org
.apache
.hadoop
.hbase
.KeyValue
;
81 import org
.apache
.hadoop
.hbase
.MasterNotRunningException
;
82 import org
.apache
.hadoop
.hbase
.MetaTableAccessor
;
83 import org
.apache
.hadoop
.hbase
.RegionLocations
;
84 import org
.apache
.hadoop
.hbase
.ServerName
;
85 import org
.apache
.hadoop
.hbase
.TableName
;
86 import org
.apache
.hadoop
.hbase
.TableNotFoundException
;
87 import org
.apache
.hadoop
.hbase
.ZooKeeperConnectionException
;
88 import org
.apache
.hadoop
.hbase
.client
.Admin
;
89 import org
.apache
.hadoop
.hbase
.client
.ColumnFamilyDescriptor
;
90 import org
.apache
.hadoop
.hbase
.client
.ColumnFamilyDescriptorBuilder
;
91 import org
.apache
.hadoop
.hbase
.client
.Connection
;
92 import org
.apache
.hadoop
.hbase
.client
.ConnectionFactory
;
93 import org
.apache
.hadoop
.hbase
.client
.Delete
;
94 import org
.apache
.hadoop
.hbase
.client
.Get
;
95 import org
.apache
.hadoop
.hbase
.client
.Put
;
96 import org
.apache
.hadoop
.hbase
.client
.RegionInfo
;
97 import org
.apache
.hadoop
.hbase
.client
.RegionInfoBuilder
;
98 import org
.apache
.hadoop
.hbase
.client
.RegionLocator
;
99 import org
.apache
.hadoop
.hbase
.client
.RegionReplicaUtil
;
100 import org
.apache
.hadoop
.hbase
.client
.Result
;
101 import org
.apache
.hadoop
.hbase
.client
.ResultScanner
;
102 import org
.apache
.hadoop
.hbase
.client
.RowMutations
;
103 import org
.apache
.hadoop
.hbase
.client
.Scan
;
104 import org
.apache
.hadoop
.hbase
.client
.Table
;
105 import org
.apache
.hadoop
.hbase
.client
.TableDescriptor
;
106 import org
.apache
.hadoop
.hbase
.client
.TableDescriptorBuilder
;
107 import org
.apache
.hadoop
.hbase
.client
.TableState
;
108 import org
.apache
.hadoop
.hbase
.io
.FileLink
;
109 import org
.apache
.hadoop
.hbase
.io
.HFileLink
;
110 import org
.apache
.hadoop
.hbase
.io
.hfile
.CacheConfig
;
111 import org
.apache
.hadoop
.hbase
.io
.hfile
.HFile
;
112 import org
.apache
.hadoop
.hbase
.master
.RegionState
;
113 import org
.apache
.hadoop
.hbase
.regionserver
.HRegion
;
114 import org
.apache
.hadoop
.hbase
.regionserver
.HRegionFileSystem
;
115 import org
.apache
.hadoop
.hbase
.regionserver
.StoreFileInfo
;
116 import org
.apache
.hadoop
.hbase
.replication
.ReplicationException
;
117 import org
.apache
.hadoop
.hbase
.replication
.ReplicationPeerDescription
;
118 import org
.apache
.hadoop
.hbase
.replication
.ReplicationQueueStorage
;
119 import org
.apache
.hadoop
.hbase
.replication
.ReplicationStorageFactory
;
120 import org
.apache
.hadoop
.hbase
.security
.UserProvider
;
121 import org
.apache
.hadoop
.hbase
.util
.Bytes
.ByteArrayComparator
;
122 import org
.apache
.hadoop
.hbase
.util
.HbckErrorReporter
.ERROR_CODE
;
123 import org
.apache
.hadoop
.hbase
.util
.hbck
.HFileCorruptionChecker
;
124 import org
.apache
.hadoop
.hbase
.util
.hbck
.ReplicationChecker
;
125 import org
.apache
.hadoop
.hbase
.util
.hbck
.TableIntegrityErrorHandler
;
126 import org
.apache
.hadoop
.hbase
.wal
.WALSplitUtil
;
127 import org
.apache
.hadoop
.hbase
.zookeeper
.ZKUtil
;
128 import org
.apache
.hadoop
.hbase
.zookeeper
.ZKWatcher
;
129 import org
.apache
.hadoop
.hbase
.zookeeper
.ZNodePaths
;
130 import org
.apache
.hadoop
.hdfs
.protocol
.AlreadyBeingCreatedException
;
131 import org
.apache
.hadoop
.ipc
.RemoteException
;
132 import org
.apache
.hadoop
.security
.AccessControlException
;
133 import org
.apache
.hadoop
.security
.UserGroupInformation
;
134 import org
.apache
.hadoop
.util
.ReflectionUtils
;
135 import org
.apache
.hadoop
.util
.Tool
;
136 import org
.apache
.hadoop
.util
.ToolRunner
;
137 import org
.apache
.yetus
.audience
.InterfaceAudience
;
138 import org
.apache
.yetus
.audience
.InterfaceStability
;
139 import org
.apache
.zookeeper
.KeeperException
;
140 import org
.slf4j
.Logger
;
141 import org
.slf4j
.LoggerFactory
;
143 import org
.apache
.hbase
.thirdparty
.com
.google
.common
.annotations
.VisibleForTesting
;
144 import org
.apache
.hbase
.thirdparty
.com
.google
.common
.base
.Joiner
;
145 import org
.apache
.hbase
.thirdparty
.com
.google
.common
.base
.Preconditions
;
146 import org
.apache
.hbase
.thirdparty
.com
.google
.common
.collect
.Lists
;
147 import org
.apache
.hbase
.thirdparty
.com
.google
.common
.collect
.Sets
;
150 * HBaseFsck (hbck) is a tool for checking and repairing region consistency and
151 * table integrity problems in a corrupted HBase. This tool was written for hbase-1.x. It does not
152 * work with hbase-2.x; it can read state but is not allowed to change state; i.e. effect 'repair'.
153 * Even though it can 'read' state, given how so much has changed in how hbase1 and hbase2 operate,
154 * it will often misread. See hbck2 (HBASE-19121) for a hbck tool for hbase2. This class is
158 * Region consistency checks verify that hbase:meta, region deployment on region
159 * servers and the state of data in HDFS (.regioninfo files) all are in
162 * Table integrity checks verify that all possible row keys resolve to exactly
163 * one region of a table. This means there are no individual degenerate
164 * or backwards regions; no holes between regions; and that there are no
165 * overlapping regions.
167 * The general repair strategy works in two phases:
169 * <li> Repair Table Integrity on HDFS. (merge or fabricate regions)
170 * <li> Repair Region Consistency with hbase:meta and assignments
173 * For table integrity repairs, the tables' region directories are scanned
174 * for .regioninfo files. Each table's integrity is then verified. If there
175 * are any orphan regions (regions with no .regioninfo files) or holes, new
176 * regions are fabricated. Backwards regions are sidelined as well as empty
177 * degenerate (endkey==startkey) regions. If there are any overlapping regions,
178 * a new region is created and all data is merged into the new region.
180 * Table integrity repairs deal solely with HDFS and could potentially be done
181 * offline -- the hbase region servers or master do not need to be running.
182 * This phase can eventually be used to completely reconstruct the hbase:meta table in
183 * an offline fashion.
185 * Region consistency requires three conditions -- 1) valid .regioninfo file
186 * present in an HDFS region dir, 2) valid row with .regioninfo data in META,
187 * and 3) a region is deployed only at the regionserver that was assigned to
188 * with proper state in the master.
190 * Region consistency repairs require hbase to be online so that hbck can
191 * contact the HBase master and region servers. The hbck#connect() method must
192 * first be called successfully. Much of the region consistency information
193 * is transient and less risky to repair.
195 * If hbck is run from the command line, there are a handful of arguments that
196 * can be used to limit the kinds of repairs hbck will do. See the code in
197 * {@link #printUsageAndExit()} for more details.
198 * @deprecated For removal in hbase-4.0.0. Use HBCK2 instead.
201 @InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience
.TOOLS
)
202 @InterfaceStability.Evolving
203 public class HBaseFsck
extends Configured
implements Closeable
{
204 public static final long DEFAULT_TIME_LAG
= 60000; // default value of 1 minute
205 public static final long DEFAULT_SLEEP_BEFORE_RERUN
= 10000;
206 private static final int MAX_NUM_THREADS
= 50; // #threads to contact regions
207 private static boolean rsSupportsOffline
= true;
208 private static final int DEFAULT_OVERLAPS_TO_SIDELINE
= 2;
209 private static final int DEFAULT_MAX_MERGE
= 5;
212 * Here is where hbase-1.x used to default the lock for hbck1.
213 * It puts in place a lock when it goes to write/make changes.
216 public static final String HBCK_LOCK_FILE
= "hbase-hbck.lock";
217 private static final int DEFAULT_MAX_LOCK_FILE_ATTEMPTS
= 5;
218 private static final int DEFAULT_LOCK_FILE_ATTEMPT_SLEEP_INTERVAL
= 200; // milliseconds
219 private static final int DEFAULT_LOCK_FILE_ATTEMPT_MAX_SLEEP_TIME
= 5000; // milliseconds
220 // We have to set the timeout value > HdfsConstants.LEASE_SOFTLIMIT_PERIOD.
221 // In HADOOP-2.6 and later, the Namenode proxy now created with custom RetryPolicy for
222 // AlreadyBeingCreatedException which is implies timeout on this operations up to
223 // HdfsConstants.LEASE_SOFTLIMIT_PERIOD (60 seconds).
224 private static final int DEFAULT_WAIT_FOR_LOCK_TIMEOUT
= 80; // seconds
225 private static final int DEFAULT_MAX_CREATE_ZNODE_ATTEMPTS
= 5;
226 private static final int DEFAULT_CREATE_ZNODE_ATTEMPT_SLEEP_INTERVAL
= 200; // milliseconds
227 private static final int DEFAULT_CREATE_ZNODE_ATTEMPT_MAX_SLEEP_TIME
= 5000; // milliseconds
229 /**********************
231 **********************/
232 private static final Logger LOG
= LoggerFactory
.getLogger(HBaseFsck
.class.getName());
233 private ClusterMetrics status
;
234 private Connection connection
;
237 // threads to do ||izable tasks: retrieve data from regionservers, handle overlapping regions
238 protected ExecutorService executor
;
239 private long startMillis
= EnvironmentEdgeManager
.currentTime();
240 private HFileCorruptionChecker hfcc
;
241 private int retcode
= 0;
242 private Path HBCK_LOCK_PATH
;
243 private FSDataOutputStream hbckOutFd
;
244 // This lock is to prevent cleanup of balancer resources twice between
245 // ShutdownHook and the main code. We cleanup only if the connect() is
247 private final AtomicBoolean hbckLockCleanup
= new AtomicBoolean(false);
249 // Unsupported options in HBase 2.0+
250 private static final Set
<String
> unsupportedOptionsInV2
= Sets
.newHashSet("-fix",
251 "-fixAssignments", "-fixMeta", "-fixHdfsHoles", "-fixHdfsOrphans", "-fixTableOrphans",
252 "-fixHdfsOverlaps", "-sidelineBigOverlaps", "-fixSplitParents", "-removeParents",
253 "-fixEmptyMetaCells", "-repair", "-repairHoles", "-maxOverlapsToSideline", "-maxMerge");
258 private static boolean details
= false; // do we display the full report
259 private long timelag
= DEFAULT_TIME_LAG
; // tables whose modtime is older
260 private static boolean forceExclusive
= false; // only this hbck can modify HBase
261 private boolean fixAssignments
= false; // fix assignment errors?
262 private boolean fixMeta
= false; // fix meta errors?
263 private boolean checkHdfs
= true; // load and check fs consistency?
264 private boolean fixHdfsHoles
= false; // fix fs holes?
265 private boolean fixHdfsOverlaps
= false; // fix fs overlaps (risky)
266 private boolean fixHdfsOrphans
= false; // fix fs holes (missing .regioninfo)
267 private boolean fixTableOrphans
= false; // fix fs holes (missing .tableinfo)
268 private boolean fixVersionFile
= false; // fix missing hbase.version file in hdfs
269 private boolean fixSplitParents
= false; // fix lingering split parents
270 private boolean removeParents
= false; // remove split parents
271 private boolean fixReferenceFiles
= false; // fix lingering reference store file
272 private boolean fixHFileLinks
= false; // fix lingering HFileLinks
273 private boolean fixEmptyMetaCells
= false; // fix (remove) empty REGIONINFO_QUALIFIER rows
274 private boolean fixReplication
= false; // fix undeleted replication queues for removed peer
275 private boolean cleanReplicationBarrier
= false; // clean replication barriers of a table
276 private boolean fixAny
= false; // Set to true if any of the fix is required.
278 // limit checking/fixes to listed tables, if empty attempt to check/fix all
279 // hbase:meta are always checked
280 private Set
<TableName
> tablesIncluded
= new HashSet
<>();
281 private TableName cleanReplicationBarrierTable
;
282 private int maxMerge
= DEFAULT_MAX_MERGE
; // maximum number of overlapping regions to merge
283 // maximum number of overlapping regions to sideline
284 private int maxOverlapsToSideline
= DEFAULT_OVERLAPS_TO_SIDELINE
;
285 private boolean sidelineBigOverlaps
= false; // sideline overlaps with >maxMerge regions
286 private Path sidelineDir
= null;
288 private boolean rerun
= false; // if we tried to fix something, rerun hbck
289 private static boolean summary
= false; // if we want to print less output
290 private boolean checkMetaOnly
= false;
291 private boolean checkRegionBoundaries
= false;
292 private boolean ignorePreCheckPermission
= false; // if pre-check permission
297 final private HbckErrorReporter errors
;
301 * This map contains the state of all hbck items. It maps from encoded region
302 * name to HbckRegionInfo structure. The information contained in HbckRegionInfo is used
303 * to detect and correct consistency (hdfs/meta/deployment) problems.
305 private TreeMap
<String
, HbckRegionInfo
> regionInfoMap
= new TreeMap
<>();
306 // Empty regioninfo qualifiers in hbase:meta
307 private Set
<Result
> emptyRegionInfoQualifiers
= new HashSet
<>();
310 * This map from Tablename -> TableInfo contains the structures necessary to
311 * detect table consistency problems (holes, dupes, overlaps). It is sorted
314 * If tablesIncluded is empty, this map contains all tables.
315 * Otherwise, it contains only meta tables and tables in tablesIncluded,
316 * unless checkMetaOnly is specified, in which case, it contains only
319 private SortedMap
<TableName
, HbckTableInfo
> tablesInfo
= new ConcurrentSkipListMap
<>();
322 * When initially looking at HDFS, we attempt to find any orphaned data.
324 private List
<HbckRegionInfo
> orphanHdfsDirs
= Collections
.synchronizedList(new ArrayList
<>());
326 private Map
<TableName
, Set
<String
>> orphanTableDirs
= new HashMap
<>();
327 private Map
<TableName
, TableState
> tableStates
= new HashMap
<>();
328 private final RetryCounterFactory lockFileRetryCounterFactory
;
329 private final RetryCounterFactory createZNodeRetryCounterFactory
;
331 private Map
<TableName
, Set
<String
>> skippedRegions
= new HashMap
<>();
333 private ZKWatcher zkw
= null;
334 private String hbckEphemeralNodePath
= null;
335 private boolean hbckZodeCreated
= false;
340 * @param conf Configuration object
341 * @throws MasterNotRunningException if the master is not running
342 * @throws ZooKeeperConnectionException if unable to connect to ZooKeeper
344 public HBaseFsck(Configuration conf
) throws IOException
, ClassNotFoundException
{
345 this(conf
, createThreadPool(conf
));
348 private static ExecutorService
createThreadPool(Configuration conf
) {
349 int numThreads
= conf
.getInt("hbasefsck.numthreads", MAX_NUM_THREADS
);
350 return new ScheduledThreadPoolExecutor(numThreads
, Threads
.newDaemonThreadFactory("hbasefsck"));
357 * Configuration object
358 * @throws MasterNotRunningException
359 * if the master is not running
360 * @throws ZooKeeperConnectionException
361 * if unable to connect to ZooKeeper
363 public HBaseFsck(Configuration conf
, ExecutorService exec
) throws MasterNotRunningException
,
364 ZooKeeperConnectionException
, IOException
, ClassNotFoundException
{
366 errors
= getErrorReporter(getConf());
367 this.executor
= exec
;
368 lockFileRetryCounterFactory
= createLockRetryCounterFactory(getConf());
369 createZNodeRetryCounterFactory
= createZnodeRetryCounterFactory(getConf());
370 zkw
= createZooKeeperWatcher();
374 * @return A retry counter factory configured for retrying lock file creation.
376 public static RetryCounterFactory
createLockRetryCounterFactory(Configuration conf
) {
377 return new RetryCounterFactory(
378 conf
.getInt("hbase.hbck.lockfile.attempts", DEFAULT_MAX_LOCK_FILE_ATTEMPTS
),
379 conf
.getInt("hbase.hbck.lockfile.attempt.sleep.interval",
380 DEFAULT_LOCK_FILE_ATTEMPT_SLEEP_INTERVAL
),
381 conf
.getInt("hbase.hbck.lockfile.attempt.maxsleeptime",
382 DEFAULT_LOCK_FILE_ATTEMPT_MAX_SLEEP_TIME
));
386 * @return A retry counter factory configured for retrying znode creation.
388 private static RetryCounterFactory
createZnodeRetryCounterFactory(Configuration conf
) {
389 return new RetryCounterFactory(
390 conf
.getInt("hbase.hbck.createznode.attempts", DEFAULT_MAX_CREATE_ZNODE_ATTEMPTS
),
391 conf
.getInt("hbase.hbck.createznode.attempt.sleep.interval",
392 DEFAULT_CREATE_ZNODE_ATTEMPT_SLEEP_INTERVAL
),
393 conf
.getInt("hbase.hbck.createznode.attempt.maxsleeptime",
394 DEFAULT_CREATE_ZNODE_ATTEMPT_MAX_SLEEP_TIME
));
398 * @return Return the tmp dir this tool writes too.
401 public static Path
getTmpDir(Configuration conf
) throws IOException
{
402 return new Path(CommonFSUtils
.getRootDir(conf
), HConstants
.HBASE_TEMP_DIRECTORY
);
405 private static class FileLockCallable
implements Callable
<FSDataOutputStream
> {
406 RetryCounter retryCounter
;
407 private final Configuration conf
;
408 private Path hbckLockPath
= null;
410 public FileLockCallable(Configuration conf
, RetryCounter retryCounter
) {
411 this.retryCounter
= retryCounter
;
416 * @return Will be <code>null</code> unless you call {@link #call()}
418 Path
getHbckLockPath() {
419 return this.hbckLockPath
;
423 public FSDataOutputStream
call() throws IOException
{
425 FileSystem fs
= CommonFSUtils
.getCurrentFileSystem(this.conf
);
426 FsPermission defaultPerms
=
427 CommonFSUtils
.getFilePermissions(fs
, this.conf
, HConstants
.DATA_FILE_UMASK_KEY
);
428 Path tmpDir
= getTmpDir(conf
);
429 this.hbckLockPath
= new Path(tmpDir
, HBCK_LOCK_FILE
);
431 final FSDataOutputStream out
= createFileWithRetries(fs
, this.hbckLockPath
, defaultPerms
);
432 out
.writeBytes(InetAddress
.getLocalHost().toString());
433 // Add a note into the file we write on why hbase2 is writing out an hbck1 lock file.
434 out
.writeBytes(" Written by an hbase-2.x Master to block an " +
435 "attempt by an hbase-1.x HBCK tool making modification to state. " +
436 "See 'HBCK must match HBase server version' in the hbase refguide.");
439 } catch(RemoteException e
) {
440 if(AlreadyBeingCreatedException
.class.getName().equals(e
.getClassName())){
448 private FSDataOutputStream
createFileWithRetries(final FileSystem fs
,
449 final Path hbckLockFilePath
, final FsPermission defaultPerms
)
451 IOException exception
= null;
454 return CommonFSUtils
.create(fs
, hbckLockFilePath
, defaultPerms
, false);
455 } catch (IOException ioe
) {
456 LOG
.info("Failed to create lock file " + hbckLockFilePath
.getName()
457 + ", try=" + (retryCounter
.getAttemptTimes() + 1) + " of "
458 + retryCounter
.getMaxAttempts());
459 LOG
.debug("Failed to create lock file " + hbckLockFilePath
.getName(),
463 retryCounter
.sleepUntilNextRetry();
464 } catch (InterruptedException ie
) {
465 throw (InterruptedIOException
) new InterruptedIOException(
466 "Can't create lock file " + hbckLockFilePath
.getName())
470 } while (retryCounter
.shouldRetry());
477 * This method maintains a lock using a file. If the creation fails we return null
479 * @return FSDataOutputStream object corresponding to the newly opened lock file
480 * @throws IOException if IO failure occurs
482 public static Pair
<Path
, FSDataOutputStream
> checkAndMarkRunningHbck(Configuration conf
,
483 RetryCounter retryCounter
) throws IOException
{
484 FileLockCallable callable
= new FileLockCallable(conf
, retryCounter
);
485 ExecutorService executor
= Executors
.newFixedThreadPool(1);
486 FutureTask
<FSDataOutputStream
> futureTask
= new FutureTask
<>(callable
);
487 executor
.execute(futureTask
);
488 final int timeoutInSeconds
= conf
.getInt(
489 "hbase.hbck.lockfile.maxwaittime", DEFAULT_WAIT_FOR_LOCK_TIMEOUT
);
490 FSDataOutputStream stream
= null;
492 stream
= futureTask
.get(timeoutInSeconds
, TimeUnit
.SECONDS
);
493 } catch (ExecutionException ee
) {
494 LOG
.warn("Encountered exception when opening lock file", ee
);
495 } catch (InterruptedException ie
) {
496 LOG
.warn("Interrupted when opening lock file", ie
);
497 Thread
.currentThread().interrupt();
498 } catch (TimeoutException exception
) {
499 // took too long to obtain lock
500 LOG
.warn("Took more than " + timeoutInSeconds
+ " seconds in obtaining lock");
501 futureTask
.cancel(true);
503 executor
.shutdownNow();
505 return new Pair
<Path
, FSDataOutputStream
>(callable
.getHbckLockPath(), stream
);
508 private void unlockHbck() {
509 if (isExclusive() && hbckLockCleanup
.compareAndSet(true, false)) {
510 RetryCounter retryCounter
= lockFileRetryCounterFactory
.create();
513 IOUtils
.closeQuietly(hbckOutFd
);
514 CommonFSUtils
.delete(CommonFSUtils
.getCurrentFileSystem(getConf()), HBCK_LOCK_PATH
, true);
515 LOG
.info("Finishing hbck");
517 } catch (IOException ioe
) {
518 LOG
.info("Failed to delete " + HBCK_LOCK_PATH
+ ", try="
519 + (retryCounter
.getAttemptTimes() + 1) + " of "
520 + retryCounter
.getMaxAttempts());
521 LOG
.debug("Failed to delete " + HBCK_LOCK_PATH
, ioe
);
523 retryCounter
.sleepUntilNextRetry();
524 } catch (InterruptedException ie
) {
525 Thread
.currentThread().interrupt();
526 LOG
.warn("Interrupted while deleting lock file" +
531 } while (retryCounter
.shouldRetry());
536 * To repair region consistency, one must call connect() in order to repair
539 public void connect() throws IOException
{
543 Pair
<Path
, FSDataOutputStream
> pair
=
544 checkAndMarkRunningHbck(getConf(), this.lockFileRetryCounterFactory
.create());
545 HBCK_LOCK_PATH
= pair
.getFirst();
546 this.hbckOutFd
= pair
.getSecond();
547 if (hbckOutFd
== null) {
549 LOG
.error("Another instance of hbck is fixing HBase, exiting this instance. " +
550 "[If you are sure no other instance is running, delete the lock file " +
551 HBCK_LOCK_PATH
+ " and rerun the tool]");
552 throw new IOException("Duplicate hbck - Abort");
555 // Make sure to cleanup the lock
556 hbckLockCleanup
.set(true);
560 // Add a shutdown hook to this thread, in case user tries to
561 // kill the hbck with a ctrl-c, we want to cleanup the lock so that
562 // it is available for further calls
563 Runtime
.getRuntime().addShutdownHook(new Thread() {
566 IOUtils
.closeQuietly(HBaseFsck
.this);
572 LOG
.info("Launching hbck");
574 connection
= ConnectionFactory
.createConnection(getConf());
575 admin
= connection
.getAdmin();
576 meta
= connection
.getTable(TableName
.META_TABLE_NAME
);
577 status
= admin
.getClusterMetrics(EnumSet
.of(Option
.LIVE_SERVERS
,
578 Option
.DEAD_SERVERS
, Option
.MASTER
, Option
.BACKUP_MASTERS
,
579 Option
.REGIONS_IN_TRANSITION
, Option
.HBASE_VERSION
));
583 * Get deployed regions according to the region servers.
585 private void loadDeployedRegions() throws IOException
, InterruptedException
{
586 // From the master, get a list of all known live region servers
587 Collection
<ServerName
> regionServers
= status
.getLiveServerMetrics().keySet();
588 errors
.print("Number of live region servers: " + regionServers
.size());
590 for (ServerName rsinfo
: regionServers
) {
591 errors
.print(" " + rsinfo
.getServerName());
595 // From the master, get a list of all dead region servers
596 Collection
<ServerName
> deadRegionServers
= status
.getDeadServerNames();
597 errors
.print("Number of dead region servers: " + deadRegionServers
.size());
599 for (ServerName name
: deadRegionServers
) {
600 errors
.print(" " + name
);
604 // Print the current master name and state
605 errors
.print("Master: " + status
.getMasterName());
607 // Print the list of all backup masters
608 Collection
<ServerName
> backupMasters
= status
.getBackupMasterNames();
609 errors
.print("Number of backup masters: " + backupMasters
.size());
611 for (ServerName name
: backupMasters
) {
612 errors
.print(" " + name
);
616 errors
.print("Average load: " + status
.getAverageLoad());
617 errors
.print("Number of requests: " + status
.getRequestCount());
618 errors
.print("Number of regions: " + status
.getRegionCount());
620 List
<RegionState
> rits
= status
.getRegionStatesInTransition();
621 errors
.print("Number of regions in transition: " + rits
.size());
623 for (RegionState state
: rits
) {
624 errors
.print(" " + state
.toDescriptiveString());
628 // Determine what's deployed
629 processRegionServers(regionServers
);
633 * Clear the current state of hbck.
635 private void clearState() {
636 // Make sure regionInfo is empty before starting
638 regionInfoMap
.clear();
639 emptyRegionInfoQualifiers
.clear();
643 orphanHdfsDirs
.clear();
644 skippedRegions
.clear();
648 * This repair method analyzes hbase data in hdfs and repairs it to satisfy
649 * the table integrity rules. HBase doesn't need to be online for this
652 public void offlineHdfsIntegrityRepair() throws IOException
, InterruptedException
{
653 // Initial pass to fix orphans.
654 if (shouldCheckHdfs() && (shouldFixHdfsOrphans() || shouldFixHdfsHoles()
655 || shouldFixHdfsOverlaps() || shouldFixTableOrphans())) {
656 LOG
.info("Loading regioninfos HDFS");
657 // if nothing is happening this should always complete in two iterations.
658 int maxIterations
= getConf().getInt("hbase.hbck.integrityrepair.iterations.max", 3);
661 clearState(); // clears hbck state and reset fixes to 0 and.
662 // repair what's on HDFS
663 restoreHdfsIntegrity();
664 curIter
++;// limit the number of iterations.
665 } while (fixes
> 0 && curIter
<= maxIterations
);
667 // Repairs should be done in the first iteration and verification in the second.
668 // If there are more than 2 passes, something funny has happened.
670 if (curIter
== maxIterations
) {
671 LOG
.warn("Exiting integrity repairs after max " + curIter
+ " iterations. "
672 + "Tables integrity may not be fully repaired!");
674 LOG
.info("Successfully exiting integrity repairs after " + curIter
+ " iterations");
681 * This repair method requires the cluster to be online since it contacts
682 * region servers and the masters. It makes each region's state in HDFS, in
683 * hbase:meta, and deployments consistent.
685 * @return If > 0 , number of errors detected, if < 0 there was an unrecoverable
686 * error. If 0, we have a clean hbase.
688 public int onlineConsistencyRepair() throws IOException
, KeeperException
,
689 InterruptedException
{
691 // get regions according to what is online on each RegionServer
692 loadDeployedRegions();
693 // check whether hbase:meta is deployed and online
695 // Check if hbase:meta is found only once and in the right place
696 if (!checkMetaRegion()) {
697 String errorMsg
= "hbase:meta table is not consistent. ";
698 if (shouldFixAssignments()) {
699 errorMsg
+= "HBCK will try fixing it. Rerun once hbase:meta is back to consistent state.";
701 errorMsg
+= "Run HBCK with proper fix options to fix hbase:meta inconsistency.";
703 errors
.reportError(errorMsg
+ " Exiting...");
706 // Not going with further consistency check for tables when hbase:meta itself is not consistent.
707 LOG
.info("Loading regionsinfo from the hbase:meta table");
708 boolean success
= loadMetaEntries();
709 if (!success
) return -1;
711 // Empty cells in hbase:meta?
712 reportEmptyMetaCells();
714 // Check if we have to cleanup empty REGIONINFO_QUALIFIER rows from hbase:meta
715 if (shouldFixEmptyMetaCells()) {
719 // get a list of all tables that have not changed recently.
720 if (!checkMetaOnly
) {
721 reportTablesInFlux();
724 // Get disabled tables states
727 // load regiondirs and regioninfos from HDFS
728 if (shouldCheckHdfs()) {
729 LOG
.info("Loading region directories from HDFS");
730 loadHdfsRegionDirs();
731 LOG
.info("Loading region information from HDFS");
732 loadHdfsRegionInfos();
735 // fix the orphan tables
738 LOG
.info("Checking and fixing region consistency");
739 // Check and fix consistency
740 checkAndFixConsistency();
742 // Check integrity (does not fix)
744 return errors
.getErrorList().size();
748 * This method maintains an ephemeral znode. If the creation fails we return false or throw
751 * @return true if creating znode succeeds; false otherwise
752 * @throws IOException if IO failure occurs
754 private boolean setMasterInMaintenanceMode() throws IOException
{
755 RetryCounter retryCounter
= createZNodeRetryCounterFactory
.create();
756 hbckEphemeralNodePath
= ZNodePaths
.joinZNode(
757 zkw
.getZNodePaths().masterMaintZNode
,
758 "hbck-" + Long
.toString(EnvironmentEdgeManager
.currentTime()));
761 hbckZodeCreated
= ZKUtil
.createEphemeralNodeAndWatch(zkw
, hbckEphemeralNodePath
, null);
762 if (hbckZodeCreated
) {
765 } catch (KeeperException e
) {
766 if (retryCounter
.getAttemptTimes() >= retryCounter
.getMaxAttempts()) {
767 throw new IOException("Can't create znode " + hbckEphemeralNodePath
, e
);
769 // fall through and retry
772 LOG
.warn("Fail to create znode " + hbckEphemeralNodePath
+ ", try=" +
773 (retryCounter
.getAttemptTimes() + 1) + " of " + retryCounter
.getMaxAttempts());
776 retryCounter
.sleepUntilNextRetry();
777 } catch (InterruptedException ie
) {
778 throw (InterruptedIOException
) new InterruptedIOException(
779 "Can't create znode " + hbckEphemeralNodePath
).initCause(ie
);
781 } while (retryCounter
.shouldRetry());
782 return hbckZodeCreated
;
785 private void cleanupHbckZnode() {
787 if (zkw
!= null && hbckZodeCreated
) {
788 ZKUtil
.deleteNode(zkw
, hbckEphemeralNodePath
);
789 hbckZodeCreated
= false;
791 } catch (KeeperException e
) {
793 if (!e
.code().equals(KeeperException
.Code
.NONODE
)) {
794 LOG
.warn("Delete HBCK znode " + hbckEphemeralNodePath
+ " failed ", e
);
800 * Contacts the master and prints out cluster-wide information
801 * @return 0 on success, non-zero on failure
803 public int onlineHbck()
804 throws IOException
, KeeperException
, InterruptedException
, ReplicationException
{
805 // print hbase server version
806 errors
.print("Version: " + status
.getHBaseVersion());
810 // Do offline check and repair first
811 offlineHdfsIntegrityRepair();
812 offlineReferenceFileRepair();
813 offlineHLinkFileRepair();
814 // If Master runs maintenance tasks (such as balancer, catalog janitor, etc) during online
815 // hbck, it is likely that hbck would be misled and report transient errors. Therefore, it
816 // is better to set Master into maintenance mode during online hbck.
818 if (!setMasterInMaintenanceMode()) {
819 LOG
.warn("HBCK is running while master is not in maintenance mode, you might see transient "
820 + "error. Please run HBCK multiple times to reduce the chance of transient error.");
823 onlineConsistencyRepair();
825 if (checkRegionBoundaries
) {
826 checkRegionBoundaries();
829 checkAndFixReplication();
831 cleanReplicationBarrier();
833 // Remove the hbck znode
836 // Remove the hbck lock
839 // Print table summary
840 printTableSummary(tablesInfo
);
841 return errors
.summarize();
844 public static byte[] keyOnly(byte[] b
) {
847 int rowlength
= Bytes
.toShort(b
, 0);
848 byte[] result
= new byte[rowlength
];
849 System
.arraycopy(b
, Bytes
.SIZEOF_SHORT
, result
, 0, rowlength
);
854 public void close() throws IOException
{
858 } catch (Exception io
) {
859 LOG
.warn(io
.toString(), io
);
865 IOUtils
.closeQuietly(admin
);
866 IOUtils
.closeQuietly(meta
);
867 IOUtils
.closeQuietly(connection
);
871 private static class RegionBoundariesInformation
{
872 public byte [] regionName
;
873 public byte [] metaFirstKey
;
874 public byte [] metaLastKey
;
875 public byte [] storesFirstKey
;
876 public byte [] storesLastKey
;
878 public String
toString () {
879 return "regionName=" + Bytes
.toStringBinary(regionName
) +
880 "\nmetaFirstKey=" + Bytes
.toStringBinary(metaFirstKey
) +
881 "\nmetaLastKey=" + Bytes
.toStringBinary(metaLastKey
) +
882 "\nstoresFirstKey=" + Bytes
.toStringBinary(storesFirstKey
) +
883 "\nstoresLastKey=" + Bytes
.toStringBinary(storesLastKey
);
887 public void checkRegionBoundaries() {
889 ByteArrayComparator comparator
= new ByteArrayComparator();
890 List
<RegionInfo
> regions
= MetaTableAccessor
.getAllRegions(connection
, true);
891 final RegionBoundariesInformation currentRegionBoundariesInformation
=
892 new RegionBoundariesInformation();
893 Path hbaseRoot
= CommonFSUtils
.getRootDir(getConf());
894 for (RegionInfo regionInfo
: regions
) {
895 Path tableDir
= CommonFSUtils
.getTableDir(hbaseRoot
, regionInfo
.getTable());
896 currentRegionBoundariesInformation
.regionName
= regionInfo
.getRegionName();
897 // For each region, get the start and stop key from the META and compare them to the
898 // same information from the Stores.
899 Path path
= new Path(tableDir
, regionInfo
.getEncodedName());
900 FileSystem fs
= path
.getFileSystem(getConf());
901 FileStatus
[] files
= fs
.listStatus(path
);
902 // For all the column families in this region...
903 byte[] storeFirstKey
= null;
904 byte[] storeLastKey
= null;
905 for (FileStatus file
: files
) {
906 String fileName
= file
.getPath().toString();
907 fileName
= fileName
.substring(fileName
.lastIndexOf("/") + 1);
908 if (!fileName
.startsWith(".") && !fileName
.endsWith("recovered.edits")) {
909 FileStatus
[] storeFiles
= fs
.listStatus(file
.getPath());
910 // For all the stores in this column family.
911 for (FileStatus storeFile
: storeFiles
) {
912 HFile
.Reader reader
= HFile
.createReader(fs
, storeFile
.getPath(),
913 CacheConfig
.DISABLED
, true, getConf());
914 if ((reader
.getFirstKey() != null)
915 && ((storeFirstKey
== null) || (comparator
.compare(storeFirstKey
,
916 ((KeyValue
.KeyOnlyKeyValue
) reader
.getFirstKey().get()).getKey()) > 0))) {
917 storeFirstKey
= ((KeyValue
.KeyOnlyKeyValue
)reader
.getFirstKey().get()).getKey();
919 if ((reader
.getLastKey() != null)
920 && ((storeLastKey
== null) || (comparator
.compare(storeLastKey
,
921 ((KeyValue
.KeyOnlyKeyValue
)reader
.getLastKey().get()).getKey())) < 0)) {
922 storeLastKey
= ((KeyValue
.KeyOnlyKeyValue
)reader
.getLastKey().get()).getKey();
928 currentRegionBoundariesInformation
.metaFirstKey
= regionInfo
.getStartKey();
929 currentRegionBoundariesInformation
.metaLastKey
= regionInfo
.getEndKey();
930 currentRegionBoundariesInformation
.storesFirstKey
= keyOnly(storeFirstKey
);
931 currentRegionBoundariesInformation
.storesLastKey
= keyOnly(storeLastKey
);
932 if (currentRegionBoundariesInformation
.metaFirstKey
.length
== 0)
933 currentRegionBoundariesInformation
.metaFirstKey
= null;
934 if (currentRegionBoundariesInformation
.metaLastKey
.length
== 0)
935 currentRegionBoundariesInformation
.metaLastKey
= null;
937 // For a region to be correct, we need the META start key to be smaller or equal to the
938 // smallest start key from all the stores, and the start key from the next META entry to
939 // be bigger than the last key from all the current stores. First region start key is null;
940 // Last region end key is null; some regions can be empty and not have any store.
942 boolean valid
= true;
943 // Checking start key.
944 if ((currentRegionBoundariesInformation
.storesFirstKey
!= null)
945 && (currentRegionBoundariesInformation
.metaFirstKey
!= null)) {
947 && comparator
.compare(currentRegionBoundariesInformation
.storesFirstKey
,
948 currentRegionBoundariesInformation
.metaFirstKey
) >= 0;
950 // Checking stop key.
951 if ((currentRegionBoundariesInformation
.storesLastKey
!= null)
952 && (currentRegionBoundariesInformation
.metaLastKey
!= null)) {
954 && comparator
.compare(currentRegionBoundariesInformation
.storesLastKey
,
955 currentRegionBoundariesInformation
.metaLastKey
) < 0;
958 errors
.reportError(ERROR_CODE
.BOUNDARIES_ERROR
, "Found issues with regions boundaries",
959 tablesInfo
.get(regionInfo
.getTable()));
960 LOG
.warn("Region's boundaries not aligned between stores and META for:");
961 LOG
.warn(Objects
.toString(currentRegionBoundariesInformation
));
964 } catch (IOException e
) {
965 LOG
.error(e
.toString(), e
);
970 * Iterates through the list of all orphan/invalid regiondirs.
972 private void adoptHdfsOrphans(Collection
<HbckRegionInfo
> orphanHdfsDirs
) throws IOException
{
973 for (HbckRegionInfo hi
: orphanHdfsDirs
) {
974 LOG
.info("Attempting to handle orphan hdfs dir: " + hi
.getHdfsRegionDir());
980 * Orphaned regions are regions without a .regioninfo file in them. We "adopt"
981 * these orphans by creating a new region, and moving the column families,
982 * recovered edits, WALs, into the new region dir. We determine the region
983 * startkey and endkeys by looking at all of the hfiles inside the column
984 * families to identify the min and max keys. The resulting region will
985 * likely violate table integrity but will be dealt with by merging
986 * overlapping regions.
988 @SuppressWarnings("deprecation")
989 private void adoptHdfsOrphan(HbckRegionInfo hi
) throws IOException
{
990 Path p
= hi
.getHdfsRegionDir();
991 FileSystem fs
= p
.getFileSystem(getConf());
992 FileStatus
[] dirs
= fs
.listStatus(p
);
994 LOG
.warn("Attempt to adopt orphan hdfs region skipped because no files present in " +
995 p
+ ". This dir could probably be deleted.");
999 TableName tableName
= hi
.getTableName();
1000 HbckTableInfo tableInfo
= tablesInfo
.get(tableName
);
1001 Preconditions
.checkNotNull(tableInfo
, "Table '" + tableName
+ "' not present!");
1002 TableDescriptor template
= tableInfo
.getTableDescriptor();
1004 // find min and max key values
1005 Pair
<byte[],byte[]> orphanRegionRange
= null;
1006 for (FileStatus cf
: dirs
) {
1007 String cfName
= cf
.getPath().getName();
1008 // TODO Figure out what the special dirs are
1009 if (cfName
.startsWith(".") || cfName
.equals(HConstants
.SPLIT_LOGDIR_NAME
)) continue;
1011 FileStatus
[] hfiles
= fs
.listStatus(cf
.getPath());
1012 for (FileStatus hfile
: hfiles
) {
1014 HFile
.Reader hf
= null;
1016 hf
= HFile
.createReader(fs
, hfile
.getPath(), CacheConfig
.DISABLED
, true, getConf());
1017 Optional
<Cell
> startKv
= hf
.getFirstKey();
1018 start
= CellUtil
.cloneRow(startKv
.get());
1019 Optional
<Cell
> endKv
= hf
.getLastKey();
1020 end
= CellUtil
.cloneRow(endKv
.get());
1021 } catch (IOException ioe
) {
1022 LOG
.warn("Problem reading orphan file " + hfile
+ ", skipping");
1024 } catch (NullPointerException ioe
) {
1025 LOG
.warn("Orphan file " + hfile
+ " is possibly corrupted HFile, skipping");
1033 // expand the range to include the range of all hfiles
1034 if (orphanRegionRange
== null) {
1036 orphanRegionRange
= new Pair
<>(start
, end
);
1040 // expand range only if the hfile is wider.
1041 if (Bytes
.compareTo(orphanRegionRange
.getFirst(), start
) > 0) {
1042 orphanRegionRange
.setFirst(start
);
1044 if (Bytes
.compareTo(orphanRegionRange
.getSecond(), end
) < 0 ) {
1045 orphanRegionRange
.setSecond(end
);
1050 if (orphanRegionRange
== null) {
1051 LOG
.warn("No data in dir " + p
+ ", sidelining data");
1053 sidelineRegionDir(fs
, hi
);
1056 LOG
.info("Min max keys are : [" + Bytes
.toString(orphanRegionRange
.getFirst()) + ", " +
1057 Bytes
.toString(orphanRegionRange
.getSecond()) + ")");
1059 // create new region on hdfs. move data into place.
1060 RegionInfo regionInfo
= RegionInfoBuilder
.newBuilder(template
.getTableName())
1061 .setStartKey(orphanRegionRange
.getFirst())
1062 .setEndKey(Bytes
.add(orphanRegionRange
.getSecond(), new byte[1]))
1064 LOG
.info("Creating new region : " + regionInfo
);
1065 HRegion region
= HBaseFsckRepair
.createHDFSRegionDir(getConf(), regionInfo
, template
);
1066 Path target
= region
.getRegionFileSystem().getRegionDir();
1068 // rename all the data to new region
1069 mergeRegionDirs(target
, hi
);
1074 * This method determines if there are table integrity errors in HDFS. If
1075 * there are errors and the appropriate "fix" options are enabled, the method
1076 * will first correct orphan regions making them into legit regiondirs, and
1077 * then reload to merge potentially overlapping regions.
1079 * @return number of table integrity errors found
1081 private int restoreHdfsIntegrity() throws IOException
, InterruptedException
{
1082 // Determine what's on HDFS
1083 LOG
.info("Loading HBase regioninfo from HDFS...");
1084 loadHdfsRegionDirs(); // populating regioninfo table.
1086 int errs
= errors
.getErrorList().size();
1087 // First time just get suggestions.
1088 tablesInfo
= loadHdfsRegionInfos(); // update tableInfos based on region info in fs.
1089 checkHdfsIntegrity(false, false);
1091 if (errors
.getErrorList().size() == errs
) {
1092 LOG
.info("No integrity errors. We are done with this phase. Glorious.");
1096 if (shouldFixHdfsOrphans() && orphanHdfsDirs
.size() > 0) {
1097 adoptHdfsOrphans(orphanHdfsDirs
);
1098 // TODO optimize by incrementally adding instead of reloading.
1101 // Make sure there are no holes now.
1102 if (shouldFixHdfsHoles()) {
1103 clearState(); // this also resets # fixes.
1104 loadHdfsRegionDirs();
1105 tablesInfo
= loadHdfsRegionInfos(); // update tableInfos based on region info in fs.
1106 tablesInfo
= checkHdfsIntegrity(shouldFixHdfsHoles(), false);
1109 // Now we fix overlaps
1110 if (shouldFixHdfsOverlaps()) {
1111 // second pass we fix overlaps.
1112 clearState(); // this also resets # fixes.
1113 loadHdfsRegionDirs();
1114 tablesInfo
= loadHdfsRegionInfos(); // update tableInfos based on region info in fs.
1115 tablesInfo
= checkHdfsIntegrity(false, shouldFixHdfsOverlaps());
1118 return errors
.getErrorList().size();
1122 * Scan all the store file names to find any lingering reference files,
1123 * which refer to some none-exiting files. If "fix" option is enabled,
1124 * any lingering reference file will be sidelined if found.
1126 * Lingering reference file prevents a region from opening. It has to
1127 * be fixed before a cluster can start properly.
1129 private void offlineReferenceFileRepair() throws IOException
, InterruptedException
{
1131 Configuration conf
= getConf();
1132 Path hbaseRoot
= CommonFSUtils
.getRootDir(conf
);
1133 FileSystem fs
= hbaseRoot
.getFileSystem(conf
);
1134 LOG
.info("Computing mapping of all store files");
1135 Map
<String
, Path
> allFiles
= FSUtils
.getTableStoreFilePathMap(fs
, hbaseRoot
,
1136 new FSUtils
.ReferenceFileFilter(fs
), executor
, errors
);
1138 LOG
.info("Validating mapping using HDFS state");
1139 for (Path path
: allFiles
.values()) {
1140 Path referredToFile
= StoreFileInfo
.getReferredToFile(path
);
1141 if (fs
.exists(referredToFile
)) continue; // good, expected
1143 // Found a lingering reference file
1144 errors
.reportError(ERROR_CODE
.LINGERING_REFERENCE_HFILE
,
1145 "Found lingering reference file " + path
);
1146 if (!shouldFixReferenceFiles()) continue;
1148 // Now, trying to fix it since requested
1149 boolean success
= false;
1150 String pathStr
= path
.toString();
1152 // A reference file path should be like
1153 // ${hbase.rootdir}/data/namespace/table_name/region_id/family_name/referred_file.region_name
1154 // Up 5 directories to get the root folder.
1155 // So the file will be sidelined to a similar folder structure.
1156 int index
= pathStr
.lastIndexOf(Path
.SEPARATOR_CHAR
);
1157 for (int i
= 0; index
> 0 && i
< 5; i
++) {
1158 index
= pathStr
.lastIndexOf(Path
.SEPARATOR_CHAR
, index
- 1);
1161 Path rootDir
= getSidelineDir();
1162 Path dst
= new Path(rootDir
, pathStr
.substring(index
+ 1));
1163 fs
.mkdirs(dst
.getParent());
1164 LOG
.info("Trying to sideline reference file "
1165 + path
+ " to " + dst
);
1168 success
= fs
.rename(path
, dst
);
1173 LOG
.error("Failed to sideline reference file " + path
);
1179 * Scan all the store file names to find any lingering HFileLink files,
1180 * which refer to some none-exiting files. If "fix" option is enabled,
1181 * any lingering HFileLink file will be sidelined if found.
1183 private void offlineHLinkFileRepair() throws IOException
, InterruptedException
{
1184 Configuration conf
= getConf();
1185 Path hbaseRoot
= CommonFSUtils
.getRootDir(conf
);
1186 FileSystem fs
= hbaseRoot
.getFileSystem(conf
);
1187 LOG
.info("Computing mapping of all link files");
1188 Map
<String
, Path
> allFiles
= FSUtils
1189 .getTableStoreFilePathMap(fs
, hbaseRoot
, new FSUtils
.HFileLinkFilter(), executor
, errors
);
1192 LOG
.info("Validating mapping using HDFS state");
1193 for (Path path
: allFiles
.values()) {
1194 // building HFileLink object to gather locations
1195 HFileLink actualLink
= HFileLink
.buildFromHFileLinkPattern(conf
, path
);
1196 if (actualLink
.exists(fs
)) continue; // good, expected
1198 // Found a lingering HFileLink
1199 errors
.reportError(ERROR_CODE
.LINGERING_HFILELINK
, "Found lingering HFileLink " + path
);
1200 if (!shouldFixHFileLinks()) continue;
1202 // Now, trying to fix it since requested
1205 // An HFileLink path should be like
1206 // ${hbase.rootdir}/data/namespace/table_name/region_id/family_name/linkedtable=linkedregionname-linkedhfilename
1207 // sidelineing will happen in the ${hbase.rootdir}/${sidelinedir} directory with the same folder structure.
1208 boolean success
= sidelineFile(fs
, hbaseRoot
, path
);
1211 LOG
.error("Failed to sideline HFileLink file " + path
);
1214 // An HFileLink backreference path should be like
1215 // ${hbase.rootdir}/archive/data/namespace/table_name/region_id/family_name/.links-linkedhfilename
1216 // sidelineing will happen in the ${hbase.rootdir}/${sidelinedir} directory with the same folder structure.
1217 Path backRefPath
= FileLink
.getBackReferencesDir(HFileArchiveUtil
1218 .getStoreArchivePath(conf
, HFileLink
.getReferencedTableName(path
.getName().toString()),
1219 HFileLink
.getReferencedRegionName(path
.getName().toString()),
1220 path
.getParent().getName()),
1221 HFileLink
.getReferencedHFileName(path
.getName().toString()));
1222 success
= sidelineFile(fs
, hbaseRoot
, backRefPath
);
1225 LOG
.error("Failed to sideline HFileLink backreference file " + path
);
1230 private boolean sidelineFile(FileSystem fs
, Path hbaseRoot
, Path path
) throws IOException
{
1231 URI uri
= hbaseRoot
.toUri().relativize(path
.toUri());
1232 if (uri
.isAbsolute()) return false;
1233 String relativePath
= uri
.getPath();
1234 Path rootDir
= getSidelineDir();
1235 Path dst
= new Path(rootDir
, relativePath
);
1236 boolean pathCreated
= fs
.mkdirs(dst
.getParent());
1238 LOG
.error("Failed to create path: " + dst
.getParent());
1241 LOG
.info("Trying to sideline file " + path
+ " to " + dst
);
1242 return fs
.rename(path
, dst
);
1246 * TODO -- need to add tests for this.
1248 private void reportEmptyMetaCells() {
1249 errors
.print("Number of empty REGIONINFO_QUALIFIER rows in hbase:meta: " +
1250 emptyRegionInfoQualifiers
.size());
1252 for (Result r
: emptyRegionInfoQualifiers
) {
1253 errors
.print(" " + r
);
1259 * TODO -- need to add tests for this.
1261 private void reportTablesInFlux() {
1262 AtomicInteger numSkipped
= new AtomicInteger(0);
1263 TableDescriptor
[] allTables
= getTables(numSkipped
);
1264 errors
.print("Number of Tables: " + allTables
.length
);
1266 if (numSkipped
.get() > 0) {
1267 errors
.detail("Number of Tables in flux: " + numSkipped
.get());
1269 for (TableDescriptor td
: allTables
) {
1270 errors
.detail(" Table: " + td
.getTableName() + "\t" +
1271 (td
.isReadOnly() ?
"ro" : "rw") + "\t" +
1272 (td
.isMetaRegion() ?
"META" : " ") + "\t" +
1273 " families: " + td
.getColumnFamilyCount());
1278 public HbckErrorReporter
getErrors() {
1283 * Populate hbi's from regionInfos loaded from file system.
1285 private SortedMap
<TableName
, HbckTableInfo
> loadHdfsRegionInfos()
1286 throws IOException
, InterruptedException
{
1287 tablesInfo
.clear(); // regenerating the data
1288 // generate region split structure
1289 Collection
<HbckRegionInfo
> hbckRegionInfos
= regionInfoMap
.values();
1291 // Parallelized read of .regioninfo files.
1292 List
<WorkItemHdfsRegionInfo
> hbis
= new ArrayList
<>(hbckRegionInfos
.size());
1293 List
<Future
<Void
>> hbiFutures
;
1295 for (HbckRegionInfo hbi
: hbckRegionInfos
) {
1296 WorkItemHdfsRegionInfo work
= new WorkItemHdfsRegionInfo(hbi
, this, errors
);
1300 // Submit and wait for completion
1301 hbiFutures
= executor
.invokeAll(hbis
);
1303 for(int i
=0; i
<hbiFutures
.size(); i
++) {
1304 WorkItemHdfsRegionInfo work
= hbis
.get(i
);
1305 Future
<Void
> f
= hbiFutures
.get(i
);
1308 } catch(ExecutionException e
) {
1309 LOG
.warn("Failed to read .regioninfo file for region " +
1310 work
.hbi
.getRegionNameAsString(), e
.getCause());
1314 Path hbaseRoot
= CommonFSUtils
.getRootDir(getConf());
1315 FileSystem fs
= hbaseRoot
.getFileSystem(getConf());
1316 // serialized table info gathering.
1317 for (HbckRegionInfo hbi
: hbckRegionInfos
) {
1319 if (hbi
.getHdfsHRI() == null) {
1325 // get table name from hdfs, populate various HBaseFsck tables.
1326 TableName tableName
= hbi
.getTableName();
1327 if (tableName
== null) {
1328 // There was an entry in hbase:meta not in the HDFS?
1329 LOG
.warn("tableName was null for: " + hbi
);
1333 HbckTableInfo modTInfo
= tablesInfo
.get(tableName
);
1334 if (modTInfo
== null) {
1335 // only executed once per table.
1336 modTInfo
= new HbckTableInfo(tableName
, this);
1337 tablesInfo
.put(tableName
, modTInfo
);
1339 TableDescriptor htd
=
1340 FSTableDescriptors
.getTableDescriptorFromFs(fs
, hbaseRoot
, tableName
);
1341 modTInfo
.htds
.add(htd
);
1342 } catch (IOException ioe
) {
1343 if (!orphanTableDirs
.containsKey(tableName
)) {
1344 LOG
.warn("Unable to read .tableinfo from " + hbaseRoot
, ioe
);
1345 //should only report once for each table
1346 errors
.reportError(ERROR_CODE
.NO_TABLEINFO_FILE
,
1347 "Unable to read .tableinfo from " + hbaseRoot
+ "/" + tableName
);
1348 Set
<String
> columns
= new HashSet
<>();
1349 orphanTableDirs
.put(tableName
, getColumnFamilyList(columns
, hbi
));
1353 if (!hbi
.isSkipChecks()) {
1354 modTInfo
.addRegionInfo(hbi
);
1358 loadTableInfosForTablesWithNoRegion();
1365 * To get the column family list according to the column family dirs
1368 * @return a set of column families
1369 * @throws IOException
1371 private Set
<String
> getColumnFamilyList(Set
<String
> columns
, HbckRegionInfo hbi
)
1372 throws IOException
{
1373 Path regionDir
= hbi
.getHdfsRegionDir();
1374 FileSystem fs
= regionDir
.getFileSystem(getConf());
1375 FileStatus
[] subDirs
= fs
.listStatus(regionDir
, new FSUtils
.FamilyDirFilter(fs
));
1376 for (FileStatus subdir
: subDirs
) {
1377 String columnfamily
= subdir
.getPath().getName();
1378 columns
.add(columnfamily
);
1384 * To fabricate a .tableinfo file with following contents<br>
1385 * 1. the correct tablename <br>
1386 * 2. the correct colfamily list<br>
1387 * 3. the default properties for both {@link TableDescriptor} and {@link ColumnFamilyDescriptor}<br>
1388 * @throws IOException
1390 private boolean fabricateTableInfo(FSTableDescriptors fstd
, TableName tableName
,
1391 Set
<String
> columns
) throws IOException
{
1392 if (columns
==null || columns
.isEmpty()) return false;
1393 TableDescriptorBuilder builder
= TableDescriptorBuilder
.newBuilder(tableName
);
1394 for (String columnfamimly
: columns
) {
1395 builder
.setColumnFamily(ColumnFamilyDescriptorBuilder
.of(columnfamimly
));
1397 fstd
.createTableDescriptor(builder
.build(), true);
1402 * To fix the empty REGIONINFO_QUALIFIER rows from hbase:meta <br>
1403 * @throws IOException
1405 public void fixEmptyMetaCells() throws IOException
{
1406 if (shouldFixEmptyMetaCells() && !emptyRegionInfoQualifiers
.isEmpty()) {
1407 LOG
.info("Trying to fix empty REGIONINFO_QUALIFIER hbase:meta rows.");
1408 for (Result region
: emptyRegionInfoQualifiers
) {
1409 deleteMetaRegion(region
.getRow());
1410 errors
.getErrorList().remove(ERROR_CODE
.EMPTY_META_CELL
);
1412 emptyRegionInfoQualifiers
.clear();
1417 * To fix orphan table by creating a .tableinfo file under tableDir <br>
1418 * 1. if TableInfo is cached, to recover the .tableinfo accordingly <br>
1419 * 2. else create a default .tableinfo file with following items<br>
1420 * 2.1 the correct tablename <br>
1421 * 2.2 the correct colfamily list<br>
1422 * 2.3 the default properties for both {@link TableDescriptor} and {@link ColumnFamilyDescriptor}<br>
1423 * @throws IOException
1425 public void fixOrphanTables() throws IOException
{
1426 if (shouldFixTableOrphans() && !orphanTableDirs
.isEmpty()) {
1428 List
<TableName
> tmpList
= new ArrayList
<>(orphanTableDirs
.keySet().size());
1429 tmpList
.addAll(orphanTableDirs
.keySet());
1430 TableDescriptor
[] htds
= getTableDescriptors(tmpList
);
1431 Iterator
<Entry
<TableName
, Set
<String
>>> iter
=
1432 orphanTableDirs
.entrySet().iterator();
1434 int numFailedCase
= 0;
1435 FSTableDescriptors fstd
= new FSTableDescriptors(getConf());
1436 while (iter
.hasNext()) {
1437 Entry
<TableName
, Set
<String
>> entry
=
1439 TableName tableName
= entry
.getKey();
1440 LOG
.info("Trying to fix orphan table error: " + tableName
);
1441 if (j
< htds
.length
) {
1442 if (tableName
.equals(htds
[j
].getTableName())) {
1443 TableDescriptor htd
= htds
[j
];
1444 LOG
.info("fixing orphan table: " + tableName
+ " from cache");
1445 fstd
.createTableDescriptor(htd
, true);
1450 if (fabricateTableInfo(fstd
, tableName
, entry
.getValue())) {
1451 LOG
.warn("fixing orphan table: " + tableName
+ " with a default .tableinfo file");
1452 LOG
.warn("Strongly recommend to modify the TableDescriptor if necessary for: " + tableName
);
1455 LOG
.error("Unable to create default .tableinfo for " + tableName
+ " while missing column family information");
1462 if (orphanTableDirs
.isEmpty()) {
1463 // all orphanTableDirs are luckily recovered
1464 // re-run doFsck after recovering the .tableinfo file
1466 LOG
.warn("Strongly recommend to re-run manually hfsck after all orphanTableDirs being fixed");
1467 } else if (numFailedCase
> 0) {
1468 LOG
.error("Failed to fix " + numFailedCase
1469 + " OrphanTables with default .tableinfo files");
1474 orphanTableDirs
.clear();
1479 * Log an appropriate message about whether or not overlapping merges are computed in parallel.
1481 private void logParallelMerge() {
1482 if (getConf().getBoolean("hbasefsck.overlap.merge.parallel", true)) {
1483 LOG
.info("Handling overlap merges in parallel. set hbasefsck.overlap.merge.parallel to" +
1484 " false to run serially.");
1486 LOG
.info("Handling overlap merges serially. set hbasefsck.overlap.merge.parallel to" +
1487 " true to run in parallel.");
1491 private SortedMap
<TableName
, HbckTableInfo
> checkHdfsIntegrity(boolean fixHoles
,
1492 boolean fixOverlaps
) throws IOException
{
1493 LOG
.info("Checking HBase region split map from HDFS data...");
1495 for (HbckTableInfo tInfo
: tablesInfo
.values()) {
1496 TableIntegrityErrorHandler handler
;
1497 if (fixHoles
|| fixOverlaps
) {
1498 handler
= tInfo
.new HDFSIntegrityFixer(tInfo
, errors
, getConf(),
1499 fixHoles
, fixOverlaps
);
1501 handler
= tInfo
.new IntegrityFixSuggester(tInfo
, errors
);
1503 if (!tInfo
.checkRegionChain(handler
)) {
1504 // should dump info as well.
1505 errors
.report("Found inconsistency in table " + tInfo
.getName());
1511 Path
getSidelineDir() throws IOException
{
1512 if (sidelineDir
== null) {
1513 Path hbaseDir
= CommonFSUtils
.getRootDir(getConf());
1514 Path hbckDir
= new Path(hbaseDir
, HConstants
.HBCK_SIDELINEDIR_NAME
);
1515 sidelineDir
= new Path(hbckDir
, hbaseDir
.getName() + "-"
1522 * Sideline a region dir (instead of deleting it)
1524 Path
sidelineRegionDir(FileSystem fs
, HbckRegionInfo hi
) throws IOException
{
1525 return sidelineRegionDir(fs
, null, hi
);
1529 * Sideline a region dir (instead of deleting it)
1531 * @param parentDir if specified, the region will be sidelined to folder like
1532 * {@literal .../parentDir/<table name>/<region name>}. The purpose is to group together
1533 * similar regions sidelined, for example, those regions should be bulk loaded back later
1534 * on. If NULL, it is ignored.
1536 Path
sidelineRegionDir(FileSystem fs
,
1537 String parentDir
, HbckRegionInfo hi
) throws IOException
{
1538 TableName tableName
= hi
.getTableName();
1539 Path regionDir
= hi
.getHdfsRegionDir();
1541 if (!fs
.exists(regionDir
)) {
1542 LOG
.warn("No previous " + regionDir
+ " exists. Continuing.");
1546 Path rootDir
= getSidelineDir();
1547 if (parentDir
!= null) {
1548 rootDir
= new Path(rootDir
, parentDir
);
1550 Path sidelineTableDir
= CommonFSUtils
.getTableDir(rootDir
, tableName
);
1551 Path sidelineRegionDir
= new Path(sidelineTableDir
, regionDir
.getName());
1552 fs
.mkdirs(sidelineRegionDir
);
1553 boolean success
= false;
1554 FileStatus
[] cfs
= fs
.listStatus(regionDir
);
1556 LOG
.info("Region dir is empty: " + regionDir
);
1558 for (FileStatus cf
: cfs
) {
1559 Path src
= cf
.getPath();
1560 Path dst
= new Path(sidelineRegionDir
, src
.getName());
1561 if (fs
.isFile(src
)) {
1563 success
= fs
.rename(src
, dst
);
1565 String msg
= "Unable to rename file " + src
+ " to " + dst
;
1567 throw new IOException(msg
);
1575 LOG
.info("Sidelining files from " + src
+ " into containing region " + dst
);
1576 // FileSystem.rename is inconsistent with directories -- if the
1577 // dst (foo/a) exists and is a dir, and the src (foo/b) is a dir,
1578 // it moves the src into the dst dir resulting in (foo/a/b). If
1579 // the dst does not exist, and the src a dir, src becomes dst. (foo/b)
1580 FileStatus
[] hfiles
= fs
.listStatus(src
);
1581 if (hfiles
!= null && hfiles
.length
> 0) {
1582 for (FileStatus hfile
: hfiles
) {
1583 success
= fs
.rename(hfile
.getPath(), dst
);
1585 String msg
= "Unable to rename file " + src
+ " to " + dst
;
1587 throw new IOException(msg
);
1591 LOG
.debug("Sideline directory contents:");
1592 debugLsr(sidelineRegionDir
);
1596 LOG
.info("Removing old region dir: " + regionDir
);
1597 success
= fs
.delete(regionDir
, true);
1599 String msg
= "Unable to delete dir " + regionDir
;
1601 throw new IOException(msg
);
1603 return sidelineRegionDir
;
1607 * Load the list of disabled tables in ZK into local set.
1608 * @throws ZooKeeperConnectionException
1609 * @throws IOException
1611 private void loadTableStates()
1612 throws IOException
{
1613 tableStates
= MetaTableAccessor
.getTableStates(connection
);
1614 // Add hbase:meta so this tool keeps working. In hbase2, meta is always enabled though it
1615 // has no entry in the table states. HBCK doesn't work right w/ hbase2 but just do this in
1617 this.tableStates
.put(TableName
.META_TABLE_NAME
,
1618 new TableState(TableName
.META_TABLE_NAME
, TableState
.State
.ENABLED
));
1622 * Check if the specified region's table is disabled.
1623 * @param tableName table to check status of
1625 boolean isTableDisabled(TableName tableName
) {
1626 return tableStates
.containsKey(tableName
)
1627 && tableStates
.get(tableName
)
1628 .inStates(TableState
.State
.DISABLED
, TableState
.State
.DISABLING
);
1632 * Scan HDFS for all regions, recording their information into
1635 public void loadHdfsRegionDirs() throws IOException
, InterruptedException
{
1636 Path rootDir
= CommonFSUtils
.getRootDir(getConf());
1637 FileSystem fs
= rootDir
.getFileSystem(getConf());
1639 // list all tables from HDFS
1640 List
<FileStatus
> tableDirs
= Lists
.newArrayList();
1642 boolean foundVersionFile
= fs
.exists(new Path(rootDir
, HConstants
.VERSION_FILE_NAME
));
1644 List
<Path
> paths
= FSUtils
.getTableDirs(fs
, rootDir
);
1645 for (Path path
: paths
) {
1646 TableName tableName
= CommonFSUtils
.getTableName(path
);
1647 if ((!checkMetaOnly
&&
1648 isTableIncluded(tableName
)) ||
1649 tableName
.equals(TableName
.META_TABLE_NAME
)) {
1650 tableDirs
.add(fs
.getFileStatus(path
));
1654 // verify that version file exists
1655 if (!foundVersionFile
) {
1656 errors
.reportError(ERROR_CODE
.NO_VERSION_FILE
,
1657 "Version file does not exist in root dir " + rootDir
);
1658 if (shouldFixVersionFile()) {
1659 LOG
.info("Trying to create a new " + HConstants
.VERSION_FILE_NAME
1662 FSUtils
.setVersion(fs
, rootDir
, getConf().getInt(
1663 HConstants
.THREAD_WAKE_FREQUENCY
, 10 * 1000), getConf().getInt(
1664 HConstants
.VERSION_FILE_WRITE_ATTEMPTS
,
1665 HConstants
.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS
));
1669 // Avoid multithreading at table-level because already multithreaded internally at
1670 // region-level. Additionally multithreading at table-level can lead to deadlock
1671 // if there are many tables in the cluster. Since there are a limited # of threads
1672 // in the executor's thread pool and if we multithread at the table-level by putting
1673 // WorkItemHdfsDir callables into the executor, then we will have some threads in the
1674 // executor tied up solely in waiting for the tables' region-level calls to complete.
1675 // If there are enough tables then there will be no actual threads in the pool left
1676 // for the region-level callables to be serviced.
1677 for (FileStatus tableDir
: tableDirs
) {
1678 LOG
.debug("Loading region dirs from " +tableDir
.getPath());
1679 WorkItemHdfsDir item
= new WorkItemHdfsDir(fs
, errors
, tableDir
);
1682 } catch (ExecutionException e
) {
1683 LOG
.warn("Could not completely load table dir " +
1684 tableDir
.getPath(), e
.getCause());
1691 * Record the location of the hbase:meta region as found in ZooKeeper.
1693 private boolean recordMetaRegion() throws IOException
{
1694 List
<HRegionLocation
> locs
;
1695 try (RegionLocator locator
= connection
.getRegionLocator(TableName
.META_TABLE_NAME
)) {
1696 locs
= locator
.getRegionLocations(HConstants
.EMPTY_START_ROW
, true);
1698 if (locs
== null || locs
.isEmpty()) {
1699 errors
.reportError(ERROR_CODE
.NULL_META_REGION
, "META region was not found in ZooKeeper");
1702 for (HRegionLocation metaLocation
: locs
) {
1703 // Check if Meta region is valid and existing
1704 if (metaLocation
== null) {
1705 errors
.reportError(ERROR_CODE
.NULL_META_REGION
, "META region location is null");
1708 if (metaLocation
.getRegion() == null) {
1709 errors
.reportError(ERROR_CODE
.NULL_META_REGION
, "META location regionInfo is null");
1712 if (metaLocation
.getHostname() == null) {
1713 errors
.reportError(ERROR_CODE
.NULL_META_REGION
, "META location hostName is null");
1716 ServerName sn
= metaLocation
.getServerName();
1717 HbckRegionInfo
.MetaEntry m
= new HbckRegionInfo
.MetaEntry(metaLocation
.getRegion(), sn
,
1718 EnvironmentEdgeManager
.currentTime());
1719 HbckRegionInfo hbckRegionInfo
= regionInfoMap
.get(metaLocation
.getRegion().getEncodedName());
1720 if (hbckRegionInfo
== null) {
1721 regionInfoMap
.put(metaLocation
.getRegion().getEncodedName(), new HbckRegionInfo(m
));
1723 hbckRegionInfo
.setMetaEntry(m
);
1729 private ZKWatcher
createZooKeeperWatcher() throws IOException
{
1730 return new ZKWatcher(getConf(), "hbase Fsck", new Abortable() {
1732 public void abort(String why
, Throwable e
) {
1738 public boolean isAborted() {
1746 * Contacts each regionserver and fetches metadata about regions.
1747 * @param regionServerList - the list of region servers to connect to
1748 * @throws IOException if a remote or network exception occurs
1750 void processRegionServers(Collection
<ServerName
> regionServerList
)
1751 throws IOException
, InterruptedException
{
1753 List
<WorkItemRegion
> workItems
= new ArrayList
<>(regionServerList
.size());
1754 List
<Future
<Void
>> workFutures
;
1756 // loop to contact each region server in parallel
1757 for (ServerName rsinfo
: regionServerList
) {
1758 workItems
.add(new WorkItemRegion(this, rsinfo
, errors
, connection
));
1761 workFutures
= executor
.invokeAll(workItems
);
1763 for(int i
=0; i
<workFutures
.size(); i
++) {
1764 WorkItemRegion item
= workItems
.get(i
);
1765 Future
<Void
> f
= workFutures
.get(i
);
1768 } catch(ExecutionException e
) {
1769 LOG
.warn("Could not process regionserver {}", item
.rsinfo
.getAddress(),
1776 * Check consistency of all regions that have been found in previous phases.
1778 private void checkAndFixConsistency()
1779 throws IOException
, KeeperException
, InterruptedException
{
1780 // Divide the checks in two phases. One for default/primary replicas and another
1781 // for the non-primary ones. Keeps code cleaner this way.
1783 List
<CheckRegionConsistencyWorkItem
> workItems
= new ArrayList
<>(regionInfoMap
.size());
1784 for (java
.util
.Map
.Entry
<String
, HbckRegionInfo
> e
: regionInfoMap
.entrySet()) {
1785 if (e
.getValue().getReplicaId() == RegionInfo
.DEFAULT_REPLICA_ID
) {
1786 workItems
.add(new CheckRegionConsistencyWorkItem(e
.getKey(), e
.getValue()));
1789 checkRegionConsistencyConcurrently(workItems
);
1791 boolean prevHdfsCheck
= shouldCheckHdfs();
1792 setCheckHdfs(false); //replicas don't have any hdfs data
1793 // Run a pass over the replicas and fix any assignment issues that exist on the currently
1794 // deployed/undeployed replicas.
1795 List
<CheckRegionConsistencyWorkItem
> replicaWorkItems
= new ArrayList
<>(regionInfoMap
.size());
1796 for (java
.util
.Map
.Entry
<String
, HbckRegionInfo
> e
: regionInfoMap
.entrySet()) {
1797 if (e
.getValue().getReplicaId() != RegionInfo
.DEFAULT_REPLICA_ID
) {
1798 replicaWorkItems
.add(new CheckRegionConsistencyWorkItem(e
.getKey(), e
.getValue()));
1801 checkRegionConsistencyConcurrently(replicaWorkItems
);
1802 setCheckHdfs(prevHdfsCheck
);
1804 // If some regions is skipped during checkRegionConsistencyConcurrently() phase, we might
1805 // not get accurate state of the hbase if continuing. The config here allows users to tune
1806 // the tolerance of number of skipped region.
1807 // TODO: evaluate the consequence to continue the hbck operation without config.
1808 int terminateThreshold
= getConf().getInt("hbase.hbck.skipped.regions.limit", 0);
1809 int numOfSkippedRegions
= skippedRegions
.size();
1810 if (numOfSkippedRegions
> 0 && numOfSkippedRegions
> terminateThreshold
) {
1811 throw new IOException(numOfSkippedRegions
1812 + " region(s) could not be checked or repaired. See logs for detail.");
1815 if (shouldCheckHdfs()) {
1816 checkAndFixTableStates();
1821 * Check consistency of all regions using mulitple threads concurrently.
1823 private void checkRegionConsistencyConcurrently(
1824 final List
<CheckRegionConsistencyWorkItem
> workItems
)
1825 throws IOException
, KeeperException
, InterruptedException
{
1826 if (workItems
.isEmpty()) {
1827 return; // nothing to check
1830 List
<Future
<Void
>> workFutures
= executor
.invokeAll(workItems
);
1831 for(Future
<Void
> f
: workFutures
) {
1834 } catch(ExecutionException e1
) {
1835 LOG
.warn("Could not check region consistency " , e1
.getCause());
1836 if (e1
.getCause() instanceof IOException
) {
1837 throw (IOException
)e1
.getCause();
1838 } else if (e1
.getCause() instanceof KeeperException
) {
1839 throw (KeeperException
)e1
.getCause();
1840 } else if (e1
.getCause() instanceof InterruptedException
) {
1841 throw (InterruptedException
)e1
.getCause();
1843 throw new IOException(e1
.getCause());
1849 class CheckRegionConsistencyWorkItem
implements Callable
<Void
> {
1850 private final String key
;
1851 private final HbckRegionInfo hbi
;
1853 CheckRegionConsistencyWorkItem(String key
, HbckRegionInfo hbi
) {
1859 public synchronized Void
call() throws Exception
{
1861 checkRegionConsistency(key
, hbi
);
1862 } catch (Exception e
) {
1863 // If the region is non-META region, skip this region and send warning/error message; if
1864 // the region is META region, we should not continue.
1865 LOG
.warn("Unable to complete check or repair the region '" + hbi
.getRegionNameAsString()
1867 if (hbi
.getHdfsHRI().isMetaRegion()) {
1870 LOG
.warn("Skip region '" + hbi
.getRegionNameAsString() + "'");
1871 addSkippedRegion(hbi
);
1877 private void addSkippedRegion(final HbckRegionInfo hbi
) {
1878 Set
<String
> skippedRegionNames
= skippedRegions
.get(hbi
.getTableName());
1879 if (skippedRegionNames
== null) {
1880 skippedRegionNames
= new HashSet
<>();
1882 skippedRegionNames
.add(hbi
.getRegionNameAsString());
1883 skippedRegions
.put(hbi
.getTableName(), skippedRegionNames
);
1887 * Check and fix table states, assumes full info available:
1889 * - empty tables loaded
1891 private void checkAndFixTableStates() throws IOException
{
1892 // first check dangling states
1893 for (Entry
<TableName
, TableState
> entry
: tableStates
.entrySet()) {
1894 TableName tableName
= entry
.getKey();
1895 TableState tableState
= entry
.getValue();
1896 HbckTableInfo tableInfo
= tablesInfo
.get(tableName
);
1897 if (isTableIncluded(tableName
)
1898 && !tableName
.isSystemTable()
1899 && tableInfo
== null) {
1901 MetaTableAccessor
.deleteTableState(connection
, tableName
);
1902 TableState state
= MetaTableAccessor
.getTableState(connection
, tableName
);
1903 if (state
!= null) {
1904 errors
.reportError(ERROR_CODE
.ORPHAN_TABLE_STATE
,
1905 tableName
+ " unable to delete dangling table state " + tableState
);
1907 } else if (!checkMetaOnly
) {
1908 // dangling table state in meta if checkMetaOnly is false. If checkMetaOnly is
1909 // true, tableInfo will be null as tablesInfo are not polulated for all tables from hdfs
1910 errors
.reportError(ERROR_CODE
.ORPHAN_TABLE_STATE
,
1911 tableName
+ " has dangling table state " + tableState
);
1915 // check that all tables have states
1916 for (TableName tableName
: tablesInfo
.keySet()) {
1917 if (isTableIncluded(tableName
) && !tableStates
.containsKey(tableName
)) {
1919 MetaTableAccessor
.updateTableState(connection
, tableName
, TableState
.State
.ENABLED
);
1920 TableState newState
= MetaTableAccessor
.getTableState(connection
, tableName
);
1921 if (newState
== null) {
1922 errors
.reportError(ERROR_CODE
.NO_TABLE_STATE
,
1923 "Unable to change state for table " + tableName
+ " in meta ");
1926 errors
.reportError(ERROR_CODE
.NO_TABLE_STATE
,
1927 tableName
+ " has no state in meta ");
1933 private void preCheckPermission() throws IOException
{
1934 if (shouldIgnorePreCheckPermission()) {
1938 Path hbaseDir
= CommonFSUtils
.getRootDir(getConf());
1939 FileSystem fs
= hbaseDir
.getFileSystem(getConf());
1940 UserProvider userProvider
= UserProvider
.instantiate(getConf());
1941 UserGroupInformation ugi
= userProvider
.getCurrent().getUGI();
1942 FileStatus
[] files
= fs
.listStatus(hbaseDir
);
1943 for (FileStatus file
: files
) {
1945 fs
.access(file
.getPath(), FsAction
.WRITE
);
1946 } catch (AccessControlException ace
) {
1947 LOG
.warn("Got AccessDeniedException when preCheckPermission ", ace
);
1948 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "Current user " + ugi
.getUserName()
1949 + " does not have write perms to " + file
.getPath()
1950 + ". Please rerun hbck as hdfs user " + file
.getOwner());
1957 * Deletes region from meta table
1959 private void deleteMetaRegion(HbckRegionInfo hi
) throws IOException
{
1960 deleteMetaRegion(hi
.getMetaEntry().getRegionInfo().getRegionName());
1964 * Deletes region from meta table
1966 private void deleteMetaRegion(byte[] metaKey
) throws IOException
{
1967 Delete d
= new Delete(metaKey
);
1969 LOG
.info("Deleted " + Bytes
.toString(metaKey
) + " from META" );
1973 * Reset the split parent region info in meta table
1975 private void resetSplitParent(HbckRegionInfo hi
) throws IOException
{
1976 RowMutations mutations
= new RowMutations(hi
.getMetaEntry().getRegionInfo().getRegionName());
1977 Delete d
= new Delete(hi
.getMetaEntry().getRegionInfo().getRegionName());
1978 d
.addColumn(HConstants
.CATALOG_FAMILY
, HConstants
.SPLITA_QUALIFIER
);
1979 d
.addColumn(HConstants
.CATALOG_FAMILY
, HConstants
.SPLITB_QUALIFIER
);
1982 RegionInfo hri
= RegionInfoBuilder
.newBuilder(hi
.getMetaEntry().getRegionInfo())
1983 .setOffline(false).setSplit(false).build();
1984 Put p
= MetaTableAccessor
.makePutFromRegionInfo(hri
, EnvironmentEdgeManager
.currentTime());
1987 meta
.mutateRow(mutations
);
1988 LOG
.info("Reset split parent " + hi
.getMetaEntry().getRegionInfo().getRegionNameAsString() +
1993 * This backwards-compatibility wrapper for permanently offlining a region
1994 * that should not be alive. If the region server does not support the
1995 * "offline" method, it will use the closest unassign method instead. This
1996 * will basically work until one attempts to disable or delete the affected
1997 * table. The problem has to do with in-memory only master state, so
1998 * restarting the HMaster or failing over to another should fix this.
2000 void offline(byte[] regionName
) throws IOException
{
2001 String regionString
= Bytes
.toStringBinary(regionName
);
2002 if (!rsSupportsOffline
) {
2004 "Using unassign region " + regionString
+ " instead of using offline method, you should" +
2005 " restart HMaster after these repairs");
2006 admin
.unassign(regionName
, true);
2010 // first time we assume the rs's supports #offline.
2012 LOG
.info("Offlining region " + regionString
);
2013 admin
.offline(regionName
);
2014 } catch (IOException ioe
) {
2015 String notFoundMsg
= "java.lang.NoSuchMethodException: " +
2016 "org.apache.hadoop.hbase.master.HMaster.offline([B)";
2017 if (ioe
.getMessage().contains(notFoundMsg
)) {
2018 LOG
.warn("Using unassign region " + regionString
+
2019 " instead of using offline method, you should" +
2020 " restart HMaster after these repairs");
2021 rsSupportsOffline
= false; // in the future just use unassign
2022 admin
.unassign(regionName
, true);
2030 * Attempts to undeploy a region from a region server based in information in
2031 * META. Any operations that modify the file system should make sure that
2032 * its corresponding region is not deployed to prevent data races.
2034 * A separate call is required to update the master in-memory region state
2035 * kept in the AssignementManager. Because disable uses this state instead of
2036 * that found in META, we can't seem to cleanly disable/delete tables that
2037 * have been hbck fixed. When used on a version of HBase that does not have
2038 * the offline ipc call exposed on the master (<0.90.5, <0.92.0) a master
2039 * restart or failover may be required.
2041 void closeRegion(HbckRegionInfo hi
) throws IOException
, InterruptedException
{
2042 if (hi
.getMetaEntry() == null && hi
.getHdfsEntry() == null) {
2043 undeployRegions(hi
);
2047 // get assignment info and hregioninfo from meta.
2048 Get get
= new Get(hi
.getRegionName());
2049 get
.addColumn(HConstants
.CATALOG_FAMILY
, HConstants
.REGIONINFO_QUALIFIER
);
2050 get
.addColumn(HConstants
.CATALOG_FAMILY
, HConstants
.SERVER_QUALIFIER
);
2051 get
.addColumn(HConstants
.CATALOG_FAMILY
, HConstants
.STARTCODE_QUALIFIER
);
2052 // also get the locations of the replicas to close if the primary region is being closed
2053 if (hi
.getReplicaId() == RegionInfo
.DEFAULT_REPLICA_ID
) {
2054 int numReplicas
= admin
.getDescriptor(hi
.getTableName()).getRegionReplication();
2055 for (int i
= 0; i
< numReplicas
; i
++) {
2056 get
.addColumn(HConstants
.CATALOG_FAMILY
, CatalogFamilyFormat
.getServerColumn(i
));
2057 get
.addColumn(HConstants
.CATALOG_FAMILY
, CatalogFamilyFormat
.getStartCodeColumn(i
));
2060 Result r
= meta
.get(get
);
2061 RegionLocations rl
= CatalogFamilyFormat
.getRegionLocations(r
);
2063 LOG
.warn("Unable to close region " + hi
.getRegionNameAsString() +
2064 " since meta does not have handle to reach it");
2067 for (HRegionLocation h
: rl
.getRegionLocations()) {
2068 ServerName serverName
= h
.getServerName();
2069 if (serverName
== null) {
2070 errors
.reportError("Unable to close region "
2071 + hi
.getRegionNameAsString() + " because meta does not "
2072 + "have handle to reach it.");
2075 RegionInfo hri
= h
.getRegion();
2077 LOG
.warn("Unable to close region " + hi
.getRegionNameAsString()
2078 + " because hbase:meta had invalid or missing "
2079 + HConstants
.CATALOG_FAMILY_STR
+ ":"
2080 + Bytes
.toString(HConstants
.REGIONINFO_QUALIFIER
)
2081 + " qualifier value.");
2084 // close the region -- close files and remove assignment
2085 HBaseFsckRepair
.closeRegionSilentlyAndWait(connection
, serverName
, hri
);
2089 private void undeployRegions(HbckRegionInfo hi
) throws IOException
, InterruptedException
{
2090 undeployRegionsForHbi(hi
);
2091 // undeploy replicas of the region (but only if the method is invoked for the primary)
2092 if (hi
.getReplicaId() != RegionInfo
.DEFAULT_REPLICA_ID
) {
2095 int numReplicas
= admin
.getDescriptor(hi
.getTableName()).getRegionReplication();
2096 for (int i
= 1; i
< numReplicas
; i
++) {
2097 if (hi
.getPrimaryHRIForDeployedReplica() == null) continue;
2098 RegionInfo hri
= RegionReplicaUtil
.getRegionInfoForReplica(
2099 hi
.getPrimaryHRIForDeployedReplica(), i
);
2100 HbckRegionInfo h
= regionInfoMap
.get(hri
.getEncodedName());
2102 undeployRegionsForHbi(h
);
2103 //set skip checks; we undeployed it, and we don't want to evaluate this anymore
2104 //in consistency checks
2105 h
.setSkipChecks(true);
2110 private void undeployRegionsForHbi(HbckRegionInfo hi
) throws IOException
, InterruptedException
{
2111 for (HbckRegionInfo
.OnlineEntry rse
: hi
.getOnlineEntries()) {
2112 LOG
.debug("Undeploy region " + rse
.getRegionInfo() + " from " + rse
.getServerName());
2115 .closeRegionSilentlyAndWait(connection
, rse
.getServerName(), rse
.getRegionInfo());
2116 offline(rse
.getRegionInfo().getRegionName());
2117 } catch (IOException ioe
) {
2118 LOG
.warn("Got exception when attempting to offline region "
2119 + Bytes
.toString(rse
.getRegionInfo().getRegionName()), ioe
);
2124 private void tryAssignmentRepair(HbckRegionInfo hbi
, String msg
) throws IOException
,
2125 KeeperException
, InterruptedException
{
2126 // If we are trying to fix the errors
2127 if (shouldFixAssignments()) {
2129 undeployRegions(hbi
);
2131 RegionInfo hri
= hbi
.getHdfsHRI();
2133 hri
= hbi
.getMetaEntry().getRegionInfo();
2135 HBaseFsckRepair
.fixUnassigned(admin
, hri
);
2136 HBaseFsckRepair
.waitUntilAssigned(admin
, hri
);
2138 // also assign replicas if needed (do it only when this call operates on a primary replica)
2139 if (hbi
.getReplicaId() != RegionInfo
.DEFAULT_REPLICA_ID
) return;
2140 int replicationCount
= admin
.getDescriptor(hri
.getTable()).getRegionReplication();
2141 for (int i
= 1; i
< replicationCount
; i
++) {
2142 hri
= RegionReplicaUtil
.getRegionInfoForReplica(hri
, i
);
2143 HbckRegionInfo h
= regionInfoMap
.get(hri
.getEncodedName());
2146 //set skip checks; we undeploy & deploy it; we don't want to evaluate this hbi anymore
2147 //in consistency checks
2148 h
.setSkipChecks(true);
2150 HBaseFsckRepair
.fixUnassigned(admin
, hri
);
2151 HBaseFsckRepair
.waitUntilAssigned(admin
, hri
);
2158 * Check a single region for consistency and correct deployment.
2160 private void checkRegionConsistency(final String key
, final HbckRegionInfo hbi
)
2161 throws IOException
, KeeperException
, InterruptedException
{
2163 if (hbi
.isSkipChecks()) return;
2164 String descriptiveName
= hbi
.toString();
2165 boolean inMeta
= hbi
.getMetaEntry() != null;
2166 // In case not checking HDFS, assume the region is on HDFS
2167 boolean inHdfs
= !shouldCheckHdfs() || hbi
.getHdfsRegionDir() != null;
2168 boolean hasMetaAssignment
= inMeta
&& hbi
.getMetaEntry().regionServer
!= null;
2169 boolean isDeployed
= !hbi
.getDeployedOn().isEmpty();
2170 boolean isMultiplyDeployed
= hbi
.getDeployedOn().size() > 1;
2171 boolean deploymentMatchesMeta
= hasMetaAssignment
&& isDeployed
&& !isMultiplyDeployed
&&
2172 hbi
.getMetaEntry().regionServer
.equals(hbi
.getDeployedOn().get(0));
2173 boolean splitParent
= inMeta
&& hbi
.getMetaEntry().getRegionInfo().isSplit() &&
2174 hbi
.getMetaEntry().getRegionInfo().isOffline();
2175 boolean shouldBeDeployed
=
2176 inMeta
&& !isTableDisabled(hbi
.getMetaEntry().getRegionInfo().getTable());
2177 boolean recentlyModified
= inHdfs
&&
2178 hbi
.getModTime() + timelag
> EnvironmentEdgeManager
.currentTime();
2180 // ========== First the healthy cases =============
2181 if (hbi
.containsOnlyHdfsEdits()) {
2184 if (inMeta
&& inHdfs
&& isDeployed
&& deploymentMatchesMeta
&& shouldBeDeployed
) {
2186 } else if (inMeta
&& inHdfs
&& !shouldBeDeployed
&& !isDeployed
) {
2187 LOG
.info("Region " + descriptiveName
+ " is in META, and in a disabled " +
2188 "tabled that is not deployed");
2190 } else if (recentlyModified
) {
2191 LOG
.warn("Region " + descriptiveName
+ " was recently modified -- skipping");
2194 // ========== Cases where the region is not in hbase:meta =============
2195 else if (!inMeta
&& !inHdfs
&& !isDeployed
) {
2196 // We shouldn't have record of this region at all then!
2197 assert false : "Entry for region with no data";
2198 } else if (!inMeta
&& !inHdfs
&& isDeployed
) {
2199 errors
.reportError(ERROR_CODE
.NOT_IN_META_HDFS
, "Region "
2200 + descriptiveName
+ ", key=" + key
+ ", not on HDFS or in hbase:meta but " +
2201 "deployed on " + Joiner
.on(", ").join(hbi
.getDeployedOn()));
2202 if (shouldFixAssignments()) {
2203 undeployRegions(hbi
);
2206 } else if (!inMeta
&& inHdfs
&& !isDeployed
) {
2207 if (hbi
.isMerged()) {
2208 // This region has already been merged, the remaining hdfs file will be
2209 // cleaned by CatalogJanitor later
2210 hbi
.setSkipChecks(true);
2211 LOG
.info("Region " + descriptiveName
2212 + " got merge recently, its file(s) will be cleaned by CatalogJanitor later");
2215 errors
.reportError(ERROR_CODE
.NOT_IN_META_OR_DEPLOYED
, "Region "
2216 + descriptiveName
+ " on HDFS, but not listed in hbase:meta " +
2217 "or deployed on any region server");
2218 // restore region consistency of an adopted orphan
2219 if (shouldFixMeta()) {
2220 if (!hbi
.isHdfsRegioninfoPresent()) {
2221 LOG
.error("Region " + hbi
.getHdfsHRI() + " could have been repaired"
2222 + " in table integrity repair phase if -fixHdfsOrphans was" +
2227 RegionInfo hri
= hbi
.getHdfsHRI();
2228 HbckTableInfo tableInfo
= tablesInfo
.get(hri
.getTable());
2230 for (RegionInfo region
: tableInfo
.getRegionsFromMeta(this.regionInfoMap
)) {
2231 if (Bytes
.compareTo(region
.getStartKey(), hri
.getStartKey()) <= 0
2232 && (region
.getEndKey().length
== 0 || Bytes
.compareTo(region
.getEndKey(),
2233 hri
.getEndKey()) >= 0)
2234 && Bytes
.compareTo(region
.getStartKey(), hri
.getEndKey()) <= 0) {
2235 if(region
.isSplit() || region
.isOffline()) continue;
2236 Path regionDir
= hbi
.getHdfsRegionDir();
2237 FileSystem fs
= regionDir
.getFileSystem(getConf());
2238 List
<Path
> familyDirs
= FSUtils
.getFamilyDirs(fs
, regionDir
);
2239 for (Path familyDir
: familyDirs
) {
2240 List
<Path
> referenceFilePaths
= FSUtils
.getReferenceFilePaths(fs
, familyDir
);
2241 for (Path referenceFilePath
: referenceFilePaths
) {
2242 Path parentRegionDir
=
2243 StoreFileInfo
.getReferredToFile(referenceFilePath
).getParent().getParent();
2244 if (parentRegionDir
.toString().endsWith(region
.getEncodedName())) {
2245 LOG
.warn(hri
+ " start and stop keys are in the range of " + region
2246 + ". The region might not be cleaned up from hdfs when region " + region
2247 + " split failed. Hence deleting from hdfs.");
2248 HRegionFileSystem
.deleteRegionFromFileSystem(getConf(), fs
,
2249 regionDir
.getParent(), hri
);
2256 LOG
.info("Patching hbase:meta with .regioninfo: " + hbi
.getHdfsHRI());
2257 int numReplicas
= admin
.getDescriptor(hbi
.getTableName()).getRegionReplication();
2258 HBaseFsckRepair
.fixMetaHoleOnlineAndAddReplicas(getConf(), hbi
.getHdfsHRI(),
2259 admin
.getClusterMetrics(EnumSet
.of(Option
.LIVE_SERVERS
))
2260 .getLiveServerMetrics().keySet(), numReplicas
);
2262 tryAssignmentRepair(hbi
, "Trying to reassign region...");
2265 } else if (!inMeta
&& inHdfs
&& isDeployed
) {
2266 errors
.reportError(ERROR_CODE
.NOT_IN_META
, "Region " + descriptiveName
2267 + " not in META, but deployed on " + Joiner
.on(", ").join(hbi
.getDeployedOn()));
2268 debugLsr(hbi
.getHdfsRegionDir());
2269 if (hbi
.getReplicaId() != RegionInfo
.DEFAULT_REPLICA_ID
) {
2270 // for replicas, this means that we should undeploy the region (we would have
2271 // gone over the primaries and fixed meta holes in first phase under
2272 // checkAndFixConsistency; we shouldn't get the condition !inMeta at
2273 // this stage unless unwanted replica)
2274 if (shouldFixAssignments()) {
2275 undeployRegionsForHbi(hbi
);
2278 if (shouldFixMeta() && hbi
.getReplicaId() == RegionInfo
.DEFAULT_REPLICA_ID
) {
2279 if (!hbi
.isHdfsRegioninfoPresent()) {
2280 LOG
.error("This should have been repaired in table integrity repair phase");
2284 LOG
.info("Patching hbase:meta with with .regioninfo: " + hbi
.getHdfsHRI());
2285 int numReplicas
= admin
.getDescriptor(hbi
.getTableName()).getRegionReplication();
2286 HBaseFsckRepair
.fixMetaHoleOnlineAndAddReplicas(getConf(), hbi
.getHdfsHRI(),
2287 admin
.getClusterMetrics(EnumSet
.of(Option
.LIVE_SERVERS
))
2288 .getLiveServerMetrics().keySet(), numReplicas
);
2289 tryAssignmentRepair(hbi
, "Trying to fix unassigned region...");
2292 // ========== Cases where the region is in hbase:meta =============
2293 } else if (inMeta
&& inHdfs
&& !isDeployed
&& splitParent
) {
2294 // check whether this is an actual error, or just transient state where parent
2296 if (hbi
.getMetaEntry().splitA
!= null && hbi
.getMetaEntry().splitB
!= null) {
2297 // check that split daughters are there
2298 HbckRegionInfo infoA
= this.regionInfoMap
.get(hbi
.getMetaEntry().splitA
.getEncodedName());
2299 HbckRegionInfo infoB
= this.regionInfoMap
.get(hbi
.getMetaEntry().splitB
.getEncodedName());
2300 if (infoA
!= null && infoB
!= null) {
2301 // we already processed or will process daughters. Move on, nothing to see here.
2302 hbi
.setSkipChecks(true);
2307 // For Replica region, we need to do a similar check. If replica is not split successfully,
2308 // error is going to be reported against primary daughter region.
2309 if (hbi
.getReplicaId() != RegionInfo
.DEFAULT_REPLICA_ID
) {
2310 LOG
.info("Region " + descriptiveName
+ " is a split parent in META, in HDFS, "
2311 + "and not deployed on any region server. This may be transient.");
2312 hbi
.setSkipChecks(true);
2316 errors
.reportError(ERROR_CODE
.LINGERING_SPLIT_PARENT
, "Region "
2317 + descriptiveName
+ " is a split parent in META, in HDFS, "
2318 + "and not deployed on any region server. This could be transient, "
2319 + "consider to run the catalog janitor first!");
2320 if (shouldFixSplitParents()) {
2322 resetSplitParent(hbi
);
2324 } else if (inMeta
&& !inHdfs
&& !isDeployed
) {
2325 errors
.reportError(ERROR_CODE
.NOT_IN_HDFS_OR_DEPLOYED
, "Region "
2326 + descriptiveName
+ " found in META, but not in HDFS "
2327 + "or deployed on any region server.");
2328 if (shouldFixMeta()) {
2329 deleteMetaRegion(hbi
);
2331 } else if (inMeta
&& !inHdfs
&& isDeployed
) {
2332 errors
.reportError(ERROR_CODE
.NOT_IN_HDFS
, "Region " + descriptiveName
2333 + " found in META, but not in HDFS, " +
2334 "and deployed on " + Joiner
.on(", ").join(hbi
.getDeployedOn()));
2335 // We treat HDFS as ground truth. Any information in meta is transient
2336 // and equivalent data can be regenerated. So, lets unassign and remove
2337 // these problems from META.
2338 if (shouldFixAssignments()) {
2339 errors
.print("Trying to fix unassigned region...");
2340 undeployRegions(hbi
);
2342 if (shouldFixMeta()) {
2343 // wait for it to complete
2344 deleteMetaRegion(hbi
);
2346 } else if (inMeta
&& inHdfs
&& !isDeployed
&& shouldBeDeployed
) {
2347 errors
.reportError(ERROR_CODE
.NOT_DEPLOYED
, "Region " + descriptiveName
2348 + " not deployed on any region server.");
2349 tryAssignmentRepair(hbi
, "Trying to fix unassigned region...");
2350 } else if (inMeta
&& inHdfs
&& isDeployed
&& !shouldBeDeployed
) {
2351 errors
.reportError(ERROR_CODE
.SHOULD_NOT_BE_DEPLOYED
,
2352 "Region " + descriptiveName
+ " should not be deployed according " +
2353 "to META, but is deployed on " + Joiner
.on(", ").join(hbi
.getDeployedOn()));
2354 if (shouldFixAssignments()) {
2355 errors
.print("Trying to close the region " + descriptiveName
);
2357 HBaseFsckRepair
.fixMultiAssignment(connection
, hbi
.getMetaEntry().getRegionInfo(),
2358 hbi
.getDeployedOn());
2360 } else if (inMeta
&& inHdfs
&& isMultiplyDeployed
) {
2361 errors
.reportError(ERROR_CODE
.MULTI_DEPLOYED
,
2362 "Region " + descriptiveName
+ " is listed in hbase:meta on region server " +
2363 hbi
.getMetaEntry().regionServer
+ " but is multiply assigned to region servers " +
2364 Joiner
.on(", ").join(hbi
.getDeployedOn()));
2365 // If we are trying to fix the errors
2366 if (shouldFixAssignments()) {
2367 errors
.print("Trying to fix assignment error...");
2369 HBaseFsckRepair
.fixMultiAssignment(connection
, hbi
.getMetaEntry().getRegionInfo(),
2370 hbi
.getDeployedOn());
2372 } else if (inMeta
&& inHdfs
&& isDeployed
&& !deploymentMatchesMeta
) {
2373 errors
.reportError(ERROR_CODE
.SERVER_DOES_NOT_MATCH_META
, "Region "
2374 + descriptiveName
+ " listed in hbase:meta on region server " +
2375 hbi
.getMetaEntry().regionServer
+ " but found on region server " +
2376 hbi
.getDeployedOn().get(0));
2377 // If we are trying to fix the errors
2378 if (shouldFixAssignments()) {
2379 errors
.print("Trying to fix assignment error...");
2381 HBaseFsckRepair
.fixMultiAssignment(connection
, hbi
.getMetaEntry().getRegionInfo(),
2382 hbi
.getDeployedOn());
2383 HBaseFsckRepair
.waitUntilAssigned(admin
, hbi
.getHdfsHRI());
2386 errors
.reportError(ERROR_CODE
.UNKNOWN
, "Region " + descriptiveName
+
2387 " is in an unforeseen state:" +
2388 " inMeta=" + inMeta
+
2389 " inHdfs=" + inHdfs
+
2390 " isDeployed=" + isDeployed
+
2391 " isMultiplyDeployed=" + isMultiplyDeployed
+
2392 " deploymentMatchesMeta=" + deploymentMatchesMeta
+
2393 " shouldBeDeployed=" + shouldBeDeployed
);
2398 * Checks tables integrity. Goes over all regions and scans the tables.
2399 * Collects all the pieces for each table and checks if there are missing,
2400 * repeated or overlapping ones.
2401 * @throws IOException
2403 SortedMap
<TableName
, HbckTableInfo
> checkIntegrity() throws IOException
{
2404 tablesInfo
= new TreeMap
<>();
2405 LOG
.debug("There are " + regionInfoMap
.size() + " region info entries");
2406 for (HbckRegionInfo hbi
: regionInfoMap
.values()) {
2407 // Check only valid, working regions
2408 if (hbi
.getMetaEntry() == null) {
2409 // this assumes that consistency check has run loadMetaEntry
2410 Path p
= hbi
.getHdfsRegionDir();
2412 errors
.report("No regioninfo in Meta or HDFS. " + hbi
);
2418 if (hbi
.getMetaEntry().regionServer
== null) {
2419 errors
.detail("Skipping region because no region server: " + hbi
);
2422 if (hbi
.getMetaEntry().getRegionInfo().isOffline()) {
2423 errors
.detail("Skipping region because it is offline: " + hbi
);
2426 if (hbi
.containsOnlyHdfsEdits()) {
2427 errors
.detail("Skipping region because it only contains edits" + hbi
);
2431 // Missing regionDir or over-deployment is checked elsewhere. Include
2432 // these cases in modTInfo, so we can evaluate those regions as part of
2433 // the region chain in META
2434 //if (hbi.foundRegionDir == null) continue;
2435 //if (hbi.deployedOn.size() != 1) continue;
2436 if (hbi
.getDeployedOn().isEmpty()) {
2440 // We should be safe here
2441 TableName tableName
= hbi
.getMetaEntry().getRegionInfo().getTable();
2442 HbckTableInfo modTInfo
= tablesInfo
.get(tableName
);
2443 if (modTInfo
== null) {
2444 modTInfo
= new HbckTableInfo(tableName
, this);
2446 for (ServerName server
: hbi
.getDeployedOn()) {
2447 modTInfo
.addServer(server
);
2450 if (!hbi
.isSkipChecks()) {
2451 modTInfo
.addRegionInfo(hbi
);
2454 tablesInfo
.put(tableName
, modTInfo
);
2457 loadTableInfosForTablesWithNoRegion();
2460 for (HbckTableInfo tInfo
: tablesInfo
.values()) {
2461 TableIntegrityErrorHandler handler
= tInfo
.new IntegrityFixSuggester(tInfo
, errors
);
2462 if (!tInfo
.checkRegionChain(handler
)) {
2463 errors
.report("Found inconsistency in table " + tInfo
.getName());
2469 /** Loads table info's for tables that may not have been included, since there are no
2470 * regions reported for the table, but table dir is there in hdfs
2472 private void loadTableInfosForTablesWithNoRegion() throws IOException
{
2473 Map
<String
, TableDescriptor
> allTables
= new FSTableDescriptors(getConf()).getAll();
2474 for (TableDescriptor htd
: allTables
.values()) {
2475 if (checkMetaOnly
&& !htd
.isMetaTable()) {
2479 TableName tableName
= htd
.getTableName();
2480 if (isTableIncluded(tableName
) && !tablesInfo
.containsKey(tableName
)) {
2481 HbckTableInfo tableInfo
= new HbckTableInfo(tableName
, this);
2482 tableInfo
.htds
.add(htd
);
2483 tablesInfo
.put(htd
.getTableName(), tableInfo
);
2489 * Merge hdfs data by moving from contained HbckRegionInfo into targetRegionDir.
2490 * @return number of file move fixes done to merge regions.
2492 public int mergeRegionDirs(Path targetRegionDir
, HbckRegionInfo contained
) throws IOException
{
2494 String thread
= Thread
.currentThread().getName();
2495 LOG
.debug("[" + thread
+ "] Contained region dir after close and pause");
2496 debugLsr(contained
.getHdfsRegionDir());
2498 // rename the contained into the container.
2499 FileSystem fs
= targetRegionDir
.getFileSystem(getConf());
2500 FileStatus
[] dirs
= null;
2502 dirs
= fs
.listStatus(contained
.getHdfsRegionDir());
2503 } catch (FileNotFoundException fnfe
) {
2504 // region we are attempting to merge in is not present! Since this is a merge, there is
2505 // no harm skipping this region if it does not exist.
2506 if (!fs
.exists(contained
.getHdfsRegionDir())) {
2507 LOG
.warn("[" + thread
+ "] HDFS region dir " + contained
.getHdfsRegionDir()
2508 + " is missing. Assuming already sidelined or moved.");
2510 sidelineRegionDir(fs
, contained
);
2516 if (!fs
.exists(contained
.getHdfsRegionDir())) {
2517 LOG
.warn("[" + thread
+ "] HDFS region dir " + contained
.getHdfsRegionDir()
2518 + " already sidelined.");
2520 sidelineRegionDir(fs
, contained
);
2525 for (FileStatus cf
: dirs
) {
2526 Path src
= cf
.getPath();
2527 Path dst
= new Path(targetRegionDir
, src
.getName());
2529 if (src
.getName().equals(HRegionFileSystem
.REGION_INFO_FILE
)) {
2530 // do not copy the old .regioninfo file.
2534 if (src
.getName().equals(HConstants
.HREGION_OLDLOGDIR_NAME
)) {
2535 // do not copy the .oldlogs files
2539 LOG
.info("[" + thread
+ "] Moving files from " + src
+ " into containing region " + dst
);
2540 // FileSystem.rename is inconsistent with directories -- if the
2541 // dst (foo/a) exists and is a dir, and the src (foo/b) is a dir,
2542 // it moves the src into the dst dir resulting in (foo/a/b). If
2543 // the dst does not exist, and the src a dir, src becomes dst. (foo/b)
2544 for (FileStatus hfile
: fs
.listStatus(src
)) {
2545 boolean success
= fs
.rename(hfile
.getPath(), dst
);
2550 LOG
.debug("[" + thread
+ "] Sideline directory contents:");
2551 debugLsr(targetRegionDir
);
2555 sidelineRegionDir(fs
, contained
);
2556 LOG
.info("[" + thread
+ "] Sidelined region dir "+ contained
.getHdfsRegionDir() + " into " +
2558 debugLsr(contained
.getHdfsRegionDir());
2564 static class WorkItemOverlapMerge
implements Callable
<Void
> {
2565 private TableIntegrityErrorHandler handler
;
2566 Collection
<HbckRegionInfo
> overlapgroup
;
2568 WorkItemOverlapMerge(Collection
<HbckRegionInfo
> overlapgroup
,
2569 TableIntegrityErrorHandler handler
) {
2570 this.handler
= handler
;
2571 this.overlapgroup
= overlapgroup
;
2575 public Void
call() throws Exception
{
2576 handler
.handleOverlapGroup(overlapgroup
);
2582 * Return a list of user-space table names whose metadata have not been
2583 * modified in the last few milliseconds specified by timelag
2584 * if any of the REGIONINFO_QUALIFIER, SERVER_QUALIFIER, STARTCODE_QUALIFIER,
2585 * SPLITA_QUALIFIER, SPLITB_QUALIFIER have not changed in the last
2586 * milliseconds specified by timelag, then the table is a candidate to be returned.
2587 * @return tables that have not been modified recently
2588 * @throws IOException if an error is encountered
2590 TableDescriptor
[] getTables(AtomicInteger numSkipped
) {
2591 List
<TableName
> tableNames
= new ArrayList
<>();
2592 long now
= EnvironmentEdgeManager
.currentTime();
2594 for (HbckRegionInfo hbi
: regionInfoMap
.values()) {
2595 HbckRegionInfo
.MetaEntry info
= hbi
.getMetaEntry();
2597 // if the start key is zero, then we have found the first region of a table.
2598 // pick only those tables that were not modified in the last few milliseconds.
2599 if (info
!= null && info
.getRegionInfo().getStartKey().length
== 0 &&
2600 !info
.getRegionInfo().isMetaRegion()) {
2601 if (info
.modTime
+ timelag
< now
) {
2602 tableNames
.add(info
.getRegionInfo().getTable());
2604 numSkipped
.incrementAndGet(); // one more in-flux table
2608 return getTableDescriptors(tableNames
);
2611 TableDescriptor
[] getTableDescriptors(List
<TableName
> tableNames
) {
2612 LOG
.info("getTableDescriptors == tableNames => " + tableNames
);
2613 try (Connection conn
= ConnectionFactory
.createConnection(getConf());
2614 Admin admin
= conn
.getAdmin()) {
2615 List
<TableDescriptor
> tds
= admin
.listTableDescriptors(tableNames
);
2616 return tds
.toArray(new TableDescriptor
[tds
.size()]);
2617 } catch (IOException e
) {
2618 LOG
.debug("Exception getting table descriptors", e
);
2620 return new TableDescriptor
[0];
2624 * Gets the entry in regionInfo corresponding to the the given encoded
2625 * region name. If the region has not been seen yet, a new entry is added
2628 private synchronized HbckRegionInfo
getOrCreateInfo(String name
) {
2629 HbckRegionInfo hbi
= regionInfoMap
.get(name
);
2631 hbi
= new HbckRegionInfo(null);
2632 regionInfoMap
.put(name
, hbi
);
2637 private void checkAndFixReplication() throws ReplicationException
{
2638 ReplicationChecker checker
= new ReplicationChecker(getConf(), zkw
, errors
);
2639 checker
.checkUnDeletedQueues();
2641 if (checker
.hasUnDeletedQueues() && this.fixReplication
) {
2642 checker
.fixUnDeletedQueues();
2648 * Check values in regionInfo for hbase:meta
2649 * Check if zero or more than one regions with hbase:meta are found.
2650 * If there are inconsistencies (i.e. zero or more than one regions
2651 * pretend to be holding the hbase:meta) try to fix that and report an error.
2652 * @throws IOException from HBaseFsckRepair functions
2653 * @throws KeeperException
2654 * @throws InterruptedException
2656 boolean checkMetaRegion() throws IOException
, KeeperException
, InterruptedException
{
2657 Map
<Integer
, HbckRegionInfo
> metaRegions
= new HashMap
<>();
2658 for (HbckRegionInfo value
: regionInfoMap
.values()) {
2659 if (value
.getMetaEntry() != null && value
.getMetaEntry().getRegionInfo().isMetaRegion()) {
2660 metaRegions
.put(value
.getReplicaId(), value
);
2663 int metaReplication
= admin
.getDescriptor(TableName
.META_TABLE_NAME
)
2664 .getRegionReplication();
2665 boolean noProblem
= true;
2666 // There will be always entries in regionInfoMap corresponding to hbase:meta & its replicas
2667 // Check the deployed servers. It should be exactly one server for each replica.
2668 for (int i
= 0; i
< metaReplication
; i
++) {
2669 HbckRegionInfo metaHbckRegionInfo
= metaRegions
.remove(i
);
2670 List
<ServerName
> servers
= new ArrayList
<>();
2671 if (metaHbckRegionInfo
!= null) {
2672 servers
= metaHbckRegionInfo
.getDeployedOn();
2674 if (servers
.size() != 1) {
2676 if (servers
.isEmpty()) {
2677 assignMetaReplica(i
);
2678 } else if (servers
.size() > 1) {
2680 .reportError(ERROR_CODE
.MULTI_META_REGION
, "hbase:meta, replicaId " +
2681 metaHbckRegionInfo
.getReplicaId() + " is found on more than one region.");
2682 if (shouldFixAssignments()) {
2683 errors
.print("Trying to fix a problem with hbase:meta, replicaId " +
2684 metaHbckRegionInfo
.getReplicaId() + "..");
2686 // try fix it (treat is a dupe assignment)
2687 HBaseFsckRepair
.fixMultiAssignment(connection
,
2688 metaHbckRegionInfo
.getMetaEntry().getRegionInfo(), servers
);
2693 // unassign whatever is remaining in metaRegions. They are excess replicas.
2694 for (Map
.Entry
<Integer
, HbckRegionInfo
> entry
: metaRegions
.entrySet()) {
2696 errors
.reportError(ERROR_CODE
.SHOULD_NOT_BE_DEPLOYED
,
2697 "hbase:meta replicas are deployed in excess. Configured " + metaReplication
+
2698 ", deployed " + metaRegions
.size());
2699 if (shouldFixAssignments()) {
2700 errors
.print("Trying to undeploy excess replica, replicaId: " + entry
.getKey() +
2701 " of hbase:meta..");
2703 unassignMetaReplica(entry
.getValue());
2706 // if noProblem is false, rerun hbck with hopefully fixed META
2707 // if noProblem is true, no errors, so continue normally
2711 private void unassignMetaReplica(HbckRegionInfo hi
)
2712 throws IOException
, InterruptedException
, KeeperException
{
2713 undeployRegions(hi
);
2714 ZKUtil
.deleteNode(zkw
,
2715 zkw
.getZNodePaths().getZNodeForReplica(hi
.getMetaEntry().getRegionInfo().getReplicaId()));
2718 private void assignMetaReplica(int replicaId
)
2719 throws IOException
, KeeperException
, InterruptedException
{
2720 errors
.reportError(ERROR_CODE
.NO_META_REGION
, "hbase:meta, replicaId " +
2721 replicaId
+" is not found on any region.");
2722 if (shouldFixAssignments()) {
2723 errors
.print("Trying to fix a problem with hbase:meta..");
2725 // try to fix it (treat it as unassigned region)
2726 RegionInfo h
= RegionReplicaUtil
.getRegionInfoForReplica(
2727 RegionInfoBuilder
.FIRST_META_REGIONINFO
, replicaId
);
2728 HBaseFsckRepair
.fixUnassigned(admin
, h
);
2729 HBaseFsckRepair
.waitUntilAssigned(admin
, h
);
2734 * Scan hbase:meta, adding all regions found to the regionInfo map.
2735 * @throws IOException if an error is encountered
2737 boolean loadMetaEntries() throws IOException
{
2738 ClientMetaTableAccessor
.Visitor visitor
= new ClientMetaTableAccessor
.Visitor() {
2739 int countRecord
= 1;
2741 // comparator to sort KeyValues with latest modtime
2742 final Comparator
<Cell
> comp
= new Comparator
<Cell
>() {
2744 public int compare(Cell k1
, Cell k2
) {
2745 return Long
.compare(k1
.getTimestamp(), k2
.getTimestamp());
2750 public boolean visit(Result result
) throws IOException
{
2753 // record the latest modification of this META record
2754 long ts
= Collections
.max(result
.listCells(), comp
).getTimestamp();
2755 RegionLocations rl
= CatalogFamilyFormat
.getRegionLocations(result
);
2757 emptyRegionInfoQualifiers
.add(result
);
2758 errors
.reportError(ERROR_CODE
.EMPTY_META_CELL
,
2759 "Empty REGIONINFO_QUALIFIER found in hbase:meta");
2762 ServerName sn
= null;
2763 if (rl
.getRegionLocation(RegionInfo
.DEFAULT_REPLICA_ID
) == null ||
2764 rl
.getRegionLocation(RegionInfo
.DEFAULT_REPLICA_ID
).getRegion() == null) {
2765 emptyRegionInfoQualifiers
.add(result
);
2766 errors
.reportError(ERROR_CODE
.EMPTY_META_CELL
,
2767 "Empty REGIONINFO_QUALIFIER found in hbase:meta");
2770 RegionInfo hri
= rl
.getRegionLocation(RegionInfo
.DEFAULT_REPLICA_ID
).getRegion();
2771 if (!(isTableIncluded(hri
.getTable())
2772 || hri
.isMetaRegion())) {
2775 PairOfSameType
<RegionInfo
> daughters
= MetaTableAccessor
.getDaughterRegions(result
);
2776 for (HRegionLocation h
: rl
.getRegionLocations()) {
2777 if (h
== null || h
.getRegion() == null) {
2780 sn
= h
.getServerName();
2781 hri
= h
.getRegion();
2783 HbckRegionInfo
.MetaEntry m
= null;
2784 if (hri
.getReplicaId() == RegionInfo
.DEFAULT_REPLICA_ID
) {
2785 m
= new HbckRegionInfo
.MetaEntry(hri
, sn
, ts
, daughters
.getFirst(),
2786 daughters
.getSecond());
2788 m
= new HbckRegionInfo
.MetaEntry(hri
, sn
, ts
, null, null);
2790 HbckRegionInfo previous
= regionInfoMap
.get(hri
.getEncodedName());
2791 if (previous
== null) {
2792 regionInfoMap
.put(hri
.getEncodedName(), new HbckRegionInfo(m
));
2793 } else if (previous
.getMetaEntry() == null) {
2794 previous
.setMetaEntry(m
);
2796 throw new IOException("Two entries in hbase:meta are same " + previous
);
2799 List
<RegionInfo
> mergeParents
= MetaTableAccessor
.getMergeRegions(result
.rawCells());
2800 if (mergeParents
!= null) {
2801 for (RegionInfo mergeRegion
: mergeParents
) {
2802 if (mergeRegion
!= null) {
2803 // This region is already being merged
2804 HbckRegionInfo hbInfo
= getOrCreateInfo(mergeRegion
.getEncodedName());
2805 hbInfo
.setMerged(true);
2810 // show proof of progress to the user, once for every 100 records.
2811 if (countRecord
% 100 == 0) {
2816 } catch (RuntimeException e
) {
2817 LOG
.error("Result=" + result
);
2822 if (!checkMetaOnly
) {
2823 // Scan hbase:meta to pick up user regions
2824 MetaTableAccessor
.fullScanRegions(connection
, visitor
);
2832 * Prints summary of all tables found on the system.
2834 private void printTableSummary(SortedMap
<TableName
, HbckTableInfo
> tablesInfo
) {
2835 StringBuilder sb
= new StringBuilder();
2836 int numOfSkippedRegions
;
2837 errors
.print("Summary:");
2838 for (HbckTableInfo tInfo
: tablesInfo
.values()) {
2839 numOfSkippedRegions
= (skippedRegions
.containsKey(tInfo
.getName())) ?
2840 skippedRegions
.get(tInfo
.getName()).size() : 0;
2842 if (errors
.tableHasErrors(tInfo
)) {
2843 errors
.print("Table " + tInfo
.getName() + " is inconsistent.");
2844 } else if (numOfSkippedRegions
> 0){
2845 errors
.print("Table " + tInfo
.getName() + " is okay (with "
2846 + numOfSkippedRegions
+ " skipped regions).");
2849 errors
.print("Table " + tInfo
.getName() + " is okay.");
2851 errors
.print(" Number of regions: " + tInfo
.getNumRegions());
2852 if (numOfSkippedRegions
> 0) {
2853 Set
<String
> skippedRegionStrings
= skippedRegions
.get(tInfo
.getName());
2854 System
.out
.println(" Number of skipped regions: " + numOfSkippedRegions
);
2855 System
.out
.println(" List of skipped regions:");
2856 for(String sr
: skippedRegionStrings
) {
2857 System
.out
.println(" " + sr
);
2860 sb
.setLength(0); // clear out existing buffer, if any.
2861 sb
.append(" Deployed on: ");
2862 for (ServerName server
: tInfo
.deployedOn
) {
2863 sb
.append(" " + server
.toString());
2865 errors
.print(sb
.toString());
2869 static HbckErrorReporter
getErrorReporter(final Configuration conf
)
2870 throws ClassNotFoundException
{
2871 Class
<?
extends HbckErrorReporter
> reporter
=
2872 conf
.getClass("hbasefsck.errorreporter", PrintingErrorReporter
.class,
2873 HbckErrorReporter
.class);
2874 return ReflectionUtils
.newInstance(reporter
, conf
);
2877 static class PrintingErrorReporter
implements HbckErrorReporter
{
2878 public int errorCount
= 0;
2879 private int showProgress
;
2880 // How frequently calls to progress() will create output
2881 private static final int progressThreshold
= 100;
2883 Set
<HbckTableInfo
> errorTables
= new HashSet
<>();
2885 // for use by unit tests to verify which errors were discovered
2886 private ArrayList
<ERROR_CODE
> errorList
= new ArrayList
<>();
2889 public void clear() {
2890 errorTables
.clear();
2896 public synchronized void reportError(ERROR_CODE errorCode
, String message
) {
2897 if (errorCode
== ERROR_CODE
.WRONG_USAGE
) {
2898 System
.err
.println(message
);
2902 errorList
.add(errorCode
);
2904 System
.out
.println("ERROR: " + message
);
2911 public synchronized void reportError(ERROR_CODE errorCode
, String message
,
2912 HbckTableInfo table
) {
2913 errorTables
.add(table
);
2914 reportError(errorCode
, message
);
2918 public synchronized void reportError(ERROR_CODE errorCode
, String message
, HbckTableInfo table
,
2919 HbckRegionInfo info
) {
2920 errorTables
.add(table
);
2921 String reference
= "(region " + info
.getRegionNameAsString() + ")";
2922 reportError(errorCode
, reference
+ " " + message
);
2926 public synchronized void reportError(ERROR_CODE errorCode
, String message
, HbckTableInfo table
,
2927 HbckRegionInfo info1
, HbckRegionInfo info2
) {
2928 errorTables
.add(table
);
2929 String reference
= "(regions " + info1
.getRegionNameAsString()
2930 + " and " + info2
.getRegionNameAsString() + ")";
2931 reportError(errorCode
, reference
+ " " + message
);
2935 public synchronized void reportError(String message
) {
2936 reportError(ERROR_CODE
.UNKNOWN
, message
);
2940 * Report error information, but do not increment the error count. Intended for cases
2941 * where the actual error would have been reported previously.
2945 public synchronized void report(String message
) {
2947 System
.out
.println("ERROR: " + message
);
2953 public synchronized int summarize() {
2954 System
.out
.println(Integer
.toString(errorCount
) +
2955 " inconsistencies detected.");
2956 if (errorCount
== 0) {
2957 System
.out
.println("Status: OK");
2960 System
.out
.println("Status: INCONSISTENT");
2966 public ArrayList
<ERROR_CODE
> getErrorList() {
2971 public synchronized void print(String message
) {
2973 System
.out
.println(message
);
2978 public boolean tableHasErrors(HbckTableInfo table
) {
2979 return errorTables
.contains(table
);
2983 public void resetErrors() {
2988 public synchronized void detail(String message
) {
2990 System
.out
.println(message
);
2996 public synchronized void progress() {
2997 if (showProgress
++ == progressThreshold
) {
2999 System
.out
.print(".");
3007 * Contact a region server and get all information from it
3009 static class WorkItemRegion
implements Callable
<Void
> {
3010 private final HBaseFsck hbck
;
3011 private final ServerName rsinfo
;
3012 private final HbckErrorReporter errors
;
3013 private final Connection connection
;
3015 WorkItemRegion(HBaseFsck hbck
, ServerName info
, HbckErrorReporter errors
,
3016 Connection connection
) {
3019 this.errors
= errors
;
3020 this.connection
= connection
;
3024 public synchronized Void
call() throws IOException
{
3027 // list all online regions from this region server
3028 List
<RegionInfo
> regions
= connection
.getAdmin().getRegions(rsinfo
);
3029 regions
= filterRegions(regions
);
3033 "RegionServer: " + rsinfo
.getServerName() + " number of regions: " + regions
.size());
3034 for (RegionInfo rinfo
: regions
) {
3035 errors
.detail(" " + rinfo
.getRegionNameAsString() + " id: " + rinfo
.getRegionId() +
3036 " encoded_name: " + rinfo
.getEncodedName() + " start: " +
3037 Bytes
.toStringBinary(rinfo
.getStartKey()) + " end: " +
3038 Bytes
.toStringBinary(rinfo
.getEndKey()));
3042 // check to see if the existence of this region matches the region in META
3043 for (RegionInfo r
: regions
) {
3044 HbckRegionInfo hbi
= hbck
.getOrCreateInfo(r
.getEncodedName());
3045 hbi
.addServer(r
, rsinfo
);
3047 } catch (IOException e
) { // unable to connect to the region server.
3048 errors
.reportError(ERROR_CODE
.RS_CONNECT_FAILURE
,
3049 "RegionServer: " + rsinfo
.getServerName() + " Unable to fetch region information. " + e
);
3055 private List
<RegionInfo
> filterRegions(List
<RegionInfo
> regions
) {
3056 List
<RegionInfo
> ret
= Lists
.newArrayList();
3057 for (RegionInfo hri
: regions
) {
3058 if (hri
.isMetaRegion() || (!hbck
.checkMetaOnly
3059 && hbck
.isTableIncluded(hri
.getTable()))) {
3068 * Contact hdfs and get all information about specified table directory into
3071 class WorkItemHdfsDir
implements Callable
<Void
> {
3072 private FileStatus tableDir
;
3073 private HbckErrorReporter errors
;
3074 private FileSystem fs
;
3076 WorkItemHdfsDir(FileSystem fs
, HbckErrorReporter errors
, FileStatus status
) {
3078 this.tableDir
= status
;
3079 this.errors
= errors
;
3083 public synchronized Void
call() throws InterruptedException
, ExecutionException
{
3084 final Vector
<Exception
> exceptions
= new Vector
<>();
3087 final FileStatus
[] regionDirs
= fs
.listStatus(tableDir
.getPath());
3088 final List
<Future
<?
>> futures
= new ArrayList
<>(regionDirs
.length
);
3090 for (final FileStatus regionDir
: regionDirs
) {
3092 final String encodedName
= regionDir
.getPath().getName();
3093 // ignore directories that aren't hexadecimal
3094 if (!encodedName
.toLowerCase(Locale
.ROOT
).matches("[0-9a-f]+")) {
3098 if (!exceptions
.isEmpty()) {
3102 futures
.add(executor
.submit(new Runnable() {
3106 LOG
.debug("Loading region info from hdfs:"+ regionDir
.getPath());
3108 Path regioninfoFile
= new Path(regionDir
.getPath(), HRegionFileSystem
.REGION_INFO_FILE
);
3109 boolean regioninfoFileExists
= fs
.exists(regioninfoFile
);
3111 if (!regioninfoFileExists
) {
3112 // As tables become larger it is more and more likely that by the time you
3113 // reach a given region that it will be gone due to region splits/merges.
3114 if (!fs
.exists(regionDir
.getPath())) {
3115 LOG
.warn("By the time we tried to process this region dir it was already gone: "
3116 + regionDir
.getPath());
3121 HbckRegionInfo hbi
= HBaseFsck
.this.getOrCreateInfo(encodedName
);
3122 HbckRegionInfo
.HdfsEntry he
= new HbckRegionInfo
.HdfsEntry();
3123 synchronized (hbi
) {
3124 if (hbi
.getHdfsRegionDir() != null) {
3125 errors
.print("Directory " + encodedName
+ " duplicate??" +
3126 hbi
.getHdfsRegionDir());
3129 he
.regionDir
= regionDir
.getPath();
3130 he
.regionDirModTime
= regionDir
.getModificationTime();
3131 he
.hdfsRegioninfoFilePresent
= regioninfoFileExists
;
3132 // we add to orphan list when we attempt to read .regioninfo
3134 // Set a flag if this region contains only edits
3135 // This is special case if a region is left after split
3136 he
.hdfsOnlyEdits
= true;
3137 FileStatus
[] subDirs
= fs
.listStatus(regionDir
.getPath());
3138 Path ePath
= WALSplitUtil
.getRegionDirRecoveredEditsDir(regionDir
.getPath());
3139 for (FileStatus subDir
: subDirs
) {
3141 String sdName
= subDir
.getPath().getName();
3142 if (!sdName
.startsWith(".") && !sdName
.equals(ePath
.getName())) {
3143 he
.hdfsOnlyEdits
= false;
3147 hbi
.setHdfsEntry(he
);
3149 } catch (Exception e
) {
3150 LOG
.error("Could not load region dir", e
);
3157 // Ensure all pending tasks are complete (or that we run into an exception)
3158 for (Future
<?
> f
: futures
) {
3159 if (!exceptions
.isEmpty()) {
3164 } catch (ExecutionException e
) {
3165 LOG
.error("Unexpected exec exception! Should've been caught already. (Bug?)", e
);
3166 // Shouldn't happen, we already logged/caught any exceptions in the Runnable
3169 } catch (IOException e
) {
3170 LOG
.error("Cannot execute WorkItemHdfsDir for " + tableDir
, e
);
3173 if (!exceptions
.isEmpty()) {
3174 errors
.reportError(ERROR_CODE
.RS_CONNECT_FAILURE
, "Table Directory: "
3175 + tableDir
.getPath().getName()
3176 + " Unable to fetch all HDFS region information. ");
3177 // Just throw the first exception as an indication something bad happened
3178 // Don't need to propagate all the exceptions, we already logged them all anyway
3179 throw new ExecutionException("First exception in WorkItemHdfsDir", exceptions
.firstElement());
3187 * Contact hdfs and get all information about specified table directory into
3190 static class WorkItemHdfsRegionInfo
implements Callable
<Void
> {
3191 private HbckRegionInfo hbi
;
3192 private HBaseFsck hbck
;
3193 private HbckErrorReporter errors
;
3195 WorkItemHdfsRegionInfo(HbckRegionInfo hbi
, HBaseFsck hbck
, HbckErrorReporter errors
) {
3198 this.errors
= errors
;
3202 public synchronized Void
call() throws IOException
{
3203 // only load entries that haven't been loaded yet.
3204 if (hbi
.getHdfsHRI() == null) {
3207 hbi
.loadHdfsRegioninfo(hbck
.getConf());
3208 } catch (IOException ioe
) {
3209 String msg
= "Orphan region in HDFS: Unable to load .regioninfo from table "
3210 + hbi
.getTableName() + " in hdfs dir "
3211 + hbi
.getHdfsRegionDir()
3212 + "! It may be an invalid format or version file. Treating as "
3213 + "an orphaned regiondir.";
3214 errors
.reportError(ERROR_CODE
.ORPHAN_HDFS_REGION
, msg
);
3216 hbck
.debugLsr(hbi
.getHdfsRegionDir());
3217 } catch (IOException ioe2
) {
3218 LOG
.error("Unable to read directory " + hbi
.getHdfsRegionDir(), ioe2
);
3221 hbck
.orphanHdfsDirs
.add(hbi
);
3230 * Display the full report from fsck. This displays all live and dead region
3231 * servers, and all known regions.
3233 public static void setDisplayFullReport() {
3237 public static boolean shouldDisplayFullReport() {
3242 * Set exclusive mode.
3244 public static void setForceExclusive() {
3245 forceExclusive
= true;
3249 * Only one instance of hbck can modify HBase at a time.
3251 public boolean isExclusive() {
3252 return fixAny
|| forceExclusive
;
3257 * Print only summary of the tables and status (OK or INCONSISTENT)
3259 static void setSummary() {
3264 * Set hbase:meta check mode.
3265 * Print only info about hbase:meta table deployment/state
3267 void setCheckMetaOnly() {
3268 checkMetaOnly
= true;
3272 * Set region boundaries check mode.
3274 void setRegionBoundariesCheck() {
3275 checkRegionBoundaries
= true;
3279 * Set replication fix mode.
3281 public void setFixReplication(boolean shouldFix
) {
3282 fixReplication
= shouldFix
;
3283 fixAny
|= shouldFix
;
3286 public void setCleanReplicationBarrier(boolean shouldClean
) {
3287 cleanReplicationBarrier
= shouldClean
;
3291 * Check if we should rerun fsck again. This checks if we've tried to
3292 * fix something and we should rerun fsck tool again.
3293 * Display the full report from fsck. This displays all live and dead
3294 * region servers, and all known regions.
3296 void setShouldRerun() {
3300 public boolean shouldRerun() {
3305 * Fix inconsistencies found by fsck. This should try to fix errors (if any)
3306 * found by fsck utility.
3308 public void setFixAssignments(boolean shouldFix
) {
3309 fixAssignments
= shouldFix
;
3310 fixAny
|= shouldFix
;
3313 boolean shouldFixAssignments() {
3314 return fixAssignments
;
3317 public void setFixMeta(boolean shouldFix
) {
3318 fixMeta
= shouldFix
;
3319 fixAny
|= shouldFix
;
3322 boolean shouldFixMeta() {
3326 public void setFixEmptyMetaCells(boolean shouldFix
) {
3327 fixEmptyMetaCells
= shouldFix
;
3328 fixAny
|= shouldFix
;
3331 boolean shouldFixEmptyMetaCells() {
3332 return fixEmptyMetaCells
;
3335 public void setCheckHdfs(boolean checking
) {
3336 checkHdfs
= checking
;
3339 boolean shouldCheckHdfs() {
3343 public void setFixHdfsHoles(boolean shouldFix
) {
3344 fixHdfsHoles
= shouldFix
;
3345 fixAny
|= shouldFix
;
3348 boolean shouldFixHdfsHoles() {
3349 return fixHdfsHoles
;
3352 public void setFixTableOrphans(boolean shouldFix
) {
3353 fixTableOrphans
= shouldFix
;
3354 fixAny
|= shouldFix
;
3357 boolean shouldFixTableOrphans() {
3358 return fixTableOrphans
;
3361 public void setFixHdfsOverlaps(boolean shouldFix
) {
3362 fixHdfsOverlaps
= shouldFix
;
3363 fixAny
|= shouldFix
;
3366 boolean shouldFixHdfsOverlaps() {
3367 return fixHdfsOverlaps
;
3370 public void setFixHdfsOrphans(boolean shouldFix
) {
3371 fixHdfsOrphans
= shouldFix
;
3372 fixAny
|= shouldFix
;
3375 boolean shouldFixHdfsOrphans() {
3376 return fixHdfsOrphans
;
3379 public void setFixVersionFile(boolean shouldFix
) {
3380 fixVersionFile
= shouldFix
;
3381 fixAny
|= shouldFix
;
3384 public boolean shouldFixVersionFile() {
3385 return fixVersionFile
;
3388 public void setSidelineBigOverlaps(boolean sbo
) {
3389 this.sidelineBigOverlaps
= sbo
;
3392 public boolean shouldSidelineBigOverlaps() {
3393 return sidelineBigOverlaps
;
3396 public void setFixSplitParents(boolean shouldFix
) {
3397 fixSplitParents
= shouldFix
;
3398 fixAny
|= shouldFix
;
3401 public void setRemoveParents(boolean shouldFix
) {
3402 removeParents
= shouldFix
;
3403 fixAny
|= shouldFix
;
3406 boolean shouldFixSplitParents() {
3407 return fixSplitParents
;
3410 boolean shouldRemoveParents() {
3411 return removeParents
;
3414 public void setFixReferenceFiles(boolean shouldFix
) {
3415 fixReferenceFiles
= shouldFix
;
3416 fixAny
|= shouldFix
;
3419 boolean shouldFixReferenceFiles() {
3420 return fixReferenceFiles
;
3423 public void setFixHFileLinks(boolean shouldFix
) {
3424 fixHFileLinks
= shouldFix
;
3425 fixAny
|= shouldFix
;
3428 boolean shouldFixHFileLinks() {
3429 return fixHFileLinks
;
3432 public boolean shouldIgnorePreCheckPermission() {
3433 return !fixAny
|| ignorePreCheckPermission
;
3436 public void setIgnorePreCheckPermission(boolean ignorePreCheckPermission
) {
3437 this.ignorePreCheckPermission
= ignorePreCheckPermission
;
3441 * @param mm maximum number of regions to merge into a single region.
3443 public void setMaxMerge(int mm
) {
3447 public int getMaxMerge() {
3451 public void setMaxOverlapsToSideline(int mo
) {
3452 this.maxOverlapsToSideline
= mo
;
3455 public int getMaxOverlapsToSideline() {
3456 return maxOverlapsToSideline
;
3460 * Only check/fix tables specified by the list,
3461 * Empty list means all tables are included.
3463 boolean isTableIncluded(TableName table
) {
3464 return (tablesIncluded
.isEmpty()) || tablesIncluded
.contains(table
);
3467 public void includeTable(TableName table
) {
3468 tablesIncluded
.add(table
);
3471 Set
<TableName
> getIncludedTables() {
3472 return new HashSet
<>(tablesIncluded
);
3476 * We are interested in only those tables that have not changed their state in
3477 * hbase:meta during the last few seconds specified by hbase.admin.fsck.timelag
3478 * @param seconds - the time in seconds
3480 public void setTimeLag(long seconds
) {
3481 timelag
= seconds
* 1000; // convert to milliseconds
3486 * @param sidelineDir - HDFS path to sideline data
3488 public void setSidelineDir(String sidelineDir
) {
3489 this.sidelineDir
= new Path(sidelineDir
);
3492 protected HFileCorruptionChecker
createHFileCorruptionChecker(boolean sidelineCorruptHFiles
) throws IOException
{
3493 return new HFileCorruptionChecker(getConf(), executor
, sidelineCorruptHFiles
);
3496 public HFileCorruptionChecker
getHFilecorruptionChecker() {
3500 public void setHFileCorruptionChecker(HFileCorruptionChecker hfcc
) {
3504 public void setRetCode(int code
) {
3505 this.retcode
= code
;
3508 public int getRetCode() {
3512 protected HBaseFsck
printUsageAndExit() {
3513 StringWriter sw
= new StringWriter(2048);
3514 PrintWriter out
= new PrintWriter(sw
);
3516 out
.println("-----------------------------------------------------------------------");
3517 out
.println("NOTE: As of HBase version 2.0, the hbck tool is significantly changed.");
3518 out
.println("In general, all Read-Only options are supported and can be be used");
3519 out
.println("safely. Most -fix/ -repair options are NOT supported. Please see usage");
3520 out
.println("below for details on which options are not supported.");
3521 out
.println("-----------------------------------------------------------------------");
3523 out
.println("Usage: fsck [opts] {only tables}");
3524 out
.println(" where [opts] are:");
3525 out
.println(" -help Display help options (this)");
3526 out
.println(" -details Display full report of all regions.");
3527 out
.println(" -timelag <timeInSeconds> Process only regions that " +
3528 " have not experienced any metadata updates in the last " +
3529 " <timeInSeconds> seconds.");
3530 out
.println(" -sleepBeforeRerun <timeInSeconds> Sleep this many seconds" +
3531 " before checking if the fix worked if run with -fix");
3532 out
.println(" -summary Print only summary of the tables and status.");
3533 out
.println(" -metaonly Only check the state of the hbase:meta table.");
3534 out
.println(" -sidelineDir <hdfs://> HDFS path to backup existing meta.");
3535 out
.println(" -boundaries Verify that regions boundaries are the same between META and store files.");
3536 out
.println(" -exclusive Abort if another hbck is exclusive or fixing.");
3539 out
.println(" Datafile Repair options: (expert features, use with caution!)");
3540 out
.println(" -checkCorruptHFiles Check all Hfiles by opening them to make sure they are valid");
3541 out
.println(" -sidelineCorruptHFiles Quarantine corrupted HFiles. implies -checkCorruptHFiles");
3544 out
.println(" Replication options");
3545 out
.println(" -fixReplication Deletes replication queues for removed peers");
3548 out
.println(" Metadata Repair options supported as of version 2.0: (expert features, use with caution!)");
3549 out
.println(" -fixVersionFile Try to fix missing hbase.version file in hdfs.");
3550 out
.println(" -fixReferenceFiles Try to offline lingering reference store files");
3551 out
.println(" -fixHFileLinks Try to offline lingering HFileLinks");
3552 out
.println(" -noHdfsChecking Don't load/check region info from HDFS."
3553 + " Assumes hbase:meta region info is good. Won't check/fix any HDFS issue, e.g. hole, orphan, or overlap");
3554 out
.println(" -ignorePreCheckPermission ignore filesystem permission pre-check");
3557 out
.println("NOTE: Following options are NOT supported as of HBase version 2.0+.");
3559 out
.println(" UNSUPPORTED Metadata Repair options: (expert features, use with caution!)");
3560 out
.println(" -fix Try to fix region assignments. This is for backwards compatiblity");
3561 out
.println(" -fixAssignments Try to fix region assignments. Replaces the old -fix");
3562 out
.println(" -fixMeta Try to fix meta problems. This assumes HDFS region info is good.");
3563 out
.println(" -fixHdfsHoles Try to fix region holes in hdfs.");
3564 out
.println(" -fixHdfsOrphans Try to fix region dirs with no .regioninfo file in hdfs");
3565 out
.println(" -fixTableOrphans Try to fix table dirs with no .tableinfo file in hdfs (online mode only)");
3566 out
.println(" -fixHdfsOverlaps Try to fix region overlaps in hdfs.");
3567 out
.println(" -maxMerge <n> When fixing region overlaps, allow at most <n> regions to merge. (n=" + DEFAULT_MAX_MERGE
+" by default)");
3568 out
.println(" -sidelineBigOverlaps When fixing region overlaps, allow to sideline big overlaps");
3569 out
.println(" -maxOverlapsToSideline <n> When fixing region overlaps, allow at most <n> regions to sideline per group. (n=" + DEFAULT_OVERLAPS_TO_SIDELINE
+" by default)");
3570 out
.println(" -fixSplitParents Try to force offline split parents to be online.");
3571 out
.println(" -removeParents Try to offline and sideline lingering parents and keep daughter regions.");
3572 out
.println(" -fixEmptyMetaCells Try to fix hbase:meta entries not referencing any region"
3573 + " (empty REGIONINFO_QUALIFIER rows)");
3576 out
.println(" UNSUPPORTED Metadata Repair shortcuts");
3577 out
.println(" -repair Shortcut for -fixAssignments -fixMeta -fixHdfsHoles " +
3578 "-fixHdfsOrphans -fixHdfsOverlaps -fixVersionFile -sidelineBigOverlaps -fixReferenceFiles" +
3580 out
.println(" -repairHoles Shortcut for -fixAssignments -fixMeta -fixHdfsHoles");
3582 out
.println(" Replication options");
3583 out
.println(" -fixReplication Deletes replication queues for removed peers");
3584 out
.println(" -cleanReplicationBrarier [tableName] clean the replication barriers " +
3585 "of a specified table, tableName is required");
3587 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, sw
.toString());
3599 public static void main(String
[] args
) throws Exception
{
3600 // create a fsck object
3601 Configuration conf
= HBaseConfiguration
.create();
3602 Path hbasedir
= CommonFSUtils
.getRootDir(conf
);
3603 URI defaultFs
= hbasedir
.getFileSystem(conf
).getUri();
3604 CommonFSUtils
.setFsDefault(conf
, new Path(defaultFs
));
3605 int ret
= ToolRunner
.run(new HBaseFsckTool(conf
), args
);
3610 * This is a Tool wrapper that gathers -Dxxx=yyy configuration settings from the command line.
3612 static class HBaseFsckTool
extends Configured
implements Tool
{
3613 HBaseFsckTool(Configuration conf
) { super(conf
); }
3615 public int run(String
[] args
) throws Exception
{
3616 HBaseFsck hbck
= new HBaseFsck(getConf());
3617 hbck
.exec(hbck
.executor
, args
);
3619 return hbck
.getRetCode();
3623 public HBaseFsck
exec(ExecutorService exec
, String
[] args
)
3624 throws KeeperException
, IOException
, InterruptedException
, ReplicationException
{
3625 long sleepBeforeRerun
= DEFAULT_SLEEP_BEFORE_RERUN
;
3627 boolean checkCorruptHFiles
= false;
3628 boolean sidelineCorruptHFiles
= false;
3630 // Process command-line args.
3631 for (int i
= 0; i
< args
.length
; i
++) {
3632 String cmd
= args
[i
];
3633 if (cmd
.equals("-help") || cmd
.equals("-h")) {
3634 return printUsageAndExit();
3635 } else if (cmd
.equals("-details")) {
3636 setDisplayFullReport();
3637 } else if (cmd
.equals("-exclusive")) {
3638 setForceExclusive();
3639 } else if (cmd
.equals("-timelag")) {
3640 if (i
== args
.length
- 1) {
3641 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "HBaseFsck: -timelag needs a value.");
3642 return printUsageAndExit();
3645 long timelag
= Long
.parseLong(args
[++i
]);
3646 setTimeLag(timelag
);
3647 } catch (NumberFormatException e
) {
3648 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "-timelag needs a numeric value.");
3649 return printUsageAndExit();
3651 } else if (cmd
.equals("-sleepBeforeRerun")) {
3652 if (i
== args
.length
- 1) {
3653 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3654 "HBaseFsck: -sleepBeforeRerun needs a value.");
3655 return printUsageAndExit();
3658 sleepBeforeRerun
= Long
.parseLong(args
[++i
]);
3659 } catch (NumberFormatException e
) {
3660 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "-sleepBeforeRerun needs a numeric value.");
3661 return printUsageAndExit();
3663 } else if (cmd
.equals("-sidelineDir")) {
3664 if (i
== args
.length
- 1) {
3665 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "HBaseFsck: -sidelineDir needs a value.");
3666 return printUsageAndExit();
3668 setSidelineDir(args
[++i
]);
3669 } else if (cmd
.equals("-fix")) {
3670 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3671 "This option is deprecated, please use -fixAssignments instead.");
3672 setFixAssignments(true);
3673 } else if (cmd
.equals("-fixAssignments")) {
3674 setFixAssignments(true);
3675 } else if (cmd
.equals("-fixMeta")) {
3677 } else if (cmd
.equals("-noHdfsChecking")) {
3678 setCheckHdfs(false);
3679 } else if (cmd
.equals("-fixHdfsHoles")) {
3680 setFixHdfsHoles(true);
3681 } else if (cmd
.equals("-fixHdfsOrphans")) {
3682 setFixHdfsOrphans(true);
3683 } else if (cmd
.equals("-fixTableOrphans")) {
3684 setFixTableOrphans(true);
3685 } else if (cmd
.equals("-fixHdfsOverlaps")) {
3686 setFixHdfsOverlaps(true);
3687 } else if (cmd
.equals("-fixVersionFile")) {
3688 setFixVersionFile(true);
3689 } else if (cmd
.equals("-sidelineBigOverlaps")) {
3690 setSidelineBigOverlaps(true);
3691 } else if (cmd
.equals("-fixSplitParents")) {
3692 setFixSplitParents(true);
3693 } else if (cmd
.equals("-removeParents")) {
3694 setRemoveParents(true);
3695 } else if (cmd
.equals("-ignorePreCheckPermission")) {
3696 setIgnorePreCheckPermission(true);
3697 } else if (cmd
.equals("-checkCorruptHFiles")) {
3698 checkCorruptHFiles
= true;
3699 } else if (cmd
.equals("-sidelineCorruptHFiles")) {
3700 sidelineCorruptHFiles
= true;
3701 } else if (cmd
.equals("-fixReferenceFiles")) {
3702 setFixReferenceFiles(true);
3703 } else if (cmd
.equals("-fixHFileLinks")) {
3704 setFixHFileLinks(true);
3705 } else if (cmd
.equals("-fixEmptyMetaCells")) {
3706 setFixEmptyMetaCells(true);
3707 } else if (cmd
.equals("-repair")) {
3708 // this attempts to merge overlapping hdfs regions, needs testing
3710 setFixHdfsHoles(true);
3711 setFixHdfsOrphans(true);
3713 setFixAssignments(true);
3714 setFixHdfsOverlaps(true);
3715 setFixVersionFile(true);
3716 setSidelineBigOverlaps(true);
3717 setFixSplitParents(false);
3719 setFixReferenceFiles(true);
3720 setFixHFileLinks(true);
3721 } else if (cmd
.equals("-repairHoles")) {
3722 // this will make all missing hdfs regions available but may lose data
3723 setFixHdfsHoles(true);
3724 setFixHdfsOrphans(false);
3726 setFixAssignments(true);
3727 setFixHdfsOverlaps(false);
3728 setSidelineBigOverlaps(false);
3729 setFixSplitParents(false);
3731 } else if (cmd
.equals("-maxOverlapsToSideline")) {
3732 if (i
== args
.length
- 1) {
3733 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3734 "-maxOverlapsToSideline needs a numeric value argument.");
3735 return printUsageAndExit();
3738 int maxOverlapsToSideline
= Integer
.parseInt(args
[++i
]);
3739 setMaxOverlapsToSideline(maxOverlapsToSideline
);
3740 } catch (NumberFormatException e
) {
3741 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3742 "-maxOverlapsToSideline needs a numeric value argument.");
3743 return printUsageAndExit();
3745 } else if (cmd
.equals("-maxMerge")) {
3746 if (i
== args
.length
- 1) {
3747 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3748 "-maxMerge needs a numeric value argument.");
3749 return printUsageAndExit();
3752 int maxMerge
= Integer
.parseInt(args
[++i
]);
3753 setMaxMerge(maxMerge
);
3754 } catch (NumberFormatException e
) {
3755 errors
.reportError(ERROR_CODE
.WRONG_USAGE
,
3756 "-maxMerge needs a numeric value argument.");
3757 return printUsageAndExit();
3759 } else if (cmd
.equals("-summary")) {
3761 } else if (cmd
.equals("-metaonly")) {
3763 } else if (cmd
.equals("-boundaries")) {
3764 setRegionBoundariesCheck();
3765 } else if (cmd
.equals("-fixReplication")) {
3766 setFixReplication(true);
3767 } else if (cmd
.equals("-cleanReplicationBarrier")) {
3768 setCleanReplicationBarrier(true);
3769 if(args
[++i
].startsWith("-")){
3770 printUsageAndExit();
3772 setCleanReplicationBarrierTable(args
[i
]);
3773 } else if (cmd
.startsWith("-")) {
3774 errors
.reportError(ERROR_CODE
.WRONG_USAGE
, "Unrecognized option:" + cmd
);
3775 return printUsageAndExit();
3777 includeTable(TableName
.valueOf(cmd
));
3778 errors
.print("Allow checking/fixes for table: " + cmd
);
3782 errors
.print("HBaseFsck command line options: " + StringUtils
.join(args
, " "));
3784 // pre-check current user has FS write permission or not
3786 preCheckPermission();
3787 } catch (IOException ioe
) {
3788 Runtime
.getRuntime().exit(-1);
3791 // do the real work of hbck
3794 // after connecting to server above, we have server version
3795 // check if unsupported option is specified based on server version
3796 if (!isOptionsSupported(args
)) {
3797 return printUsageAndExit();
3801 // if corrupt file mode is on, first fix them since they may be opened later
3802 if (checkCorruptHFiles
|| sidelineCorruptHFiles
) {
3803 LOG
.info("Checking all hfiles for corruption");
3804 HFileCorruptionChecker hfcc
= createHFileCorruptionChecker(sidelineCorruptHFiles
);
3805 setHFileCorruptionChecker(hfcc
); // so we can get result
3806 Collection
<TableName
> tables
= getIncludedTables();
3807 Collection
<Path
> tableDirs
= new ArrayList
<>();
3808 Path rootdir
= CommonFSUtils
.getRootDir(getConf());
3809 if (tables
.size() > 0) {
3810 for (TableName t
: tables
) {
3811 tableDirs
.add(CommonFSUtils
.getTableDir(rootdir
, t
));
3814 tableDirs
= FSUtils
.getTableDirs(CommonFSUtils
.getCurrentFileSystem(getConf()), rootdir
);
3816 hfcc
.checkTables(tableDirs
);
3817 hfcc
.report(errors
);
3820 // check and fix table integrity, region consistency.
3821 int code
= onlineHbck();
3823 // If we have changed the HBase state it is better to run hbck again
3824 // to see if we haven't broken something else in the process.
3825 // We run it only once more because otherwise we can easily fall into
3826 // an infinite loop.
3827 if (shouldRerun()) {
3829 LOG
.info("Sleeping " + sleepBeforeRerun
+ "ms before re-checking after fix...");
3830 Thread
.sleep(sleepBeforeRerun
);
3831 } catch (InterruptedException ie
) {
3832 LOG
.warn("Interrupted while sleeping");
3836 setFixAssignments(false);
3838 setFixHdfsHoles(false);
3839 setFixHdfsOverlaps(false);
3840 setFixVersionFile(false);
3841 setFixTableOrphans(false);
3842 errors
.resetErrors();
3843 code
= onlineHbck();
3847 IOUtils
.closeQuietly(this);
3852 private boolean isOptionsSupported(String
[] args
) {
3853 boolean result
= true;
3854 String hbaseServerVersion
= status
.getHBaseVersion();
3855 if (VersionInfo
.compareVersion("2.any.any", hbaseServerVersion
) < 0) {
3856 // Process command-line args.
3857 for (String arg
: args
) {
3858 if (unsupportedOptionsInV2
.contains(arg
)) {
3859 errors
.reportError(ERROR_CODE
.UNSUPPORTED_OPTION
,
3860 "option '" + arg
+ "' is not " + "supportted!");
3869 public void setCleanReplicationBarrierTable(String cleanReplicationBarrierTable
) {
3870 this.cleanReplicationBarrierTable
= TableName
.valueOf(cleanReplicationBarrierTable
);
3873 public void cleanReplicationBarrier() throws IOException
{
3874 if (!cleanReplicationBarrier
|| cleanReplicationBarrierTable
== null) {
3877 if (cleanReplicationBarrierTable
.isSystemTable()) {
3878 errors
.reportError(ERROR_CODE
.INVALID_TABLE
,
3879 "invalid table: " + cleanReplicationBarrierTable
);
3883 boolean isGlobalScope
= false;
3885 isGlobalScope
= admin
.getDescriptor(cleanReplicationBarrierTable
).hasGlobalReplicationScope();
3886 } catch (TableNotFoundException e
) {
3887 LOG
.info("we may need to clean some erroneous data due to bugs");
3890 if (isGlobalScope
) {
3891 errors
.reportError(ERROR_CODE
.INVALID_TABLE
,
3892 "table's replication scope is global: " + cleanReplicationBarrierTable
);
3895 List
<byte[]> regionNames
= new ArrayList
<>();
3896 Scan barrierScan
= new Scan();
3897 barrierScan
.setCaching(100);
3898 barrierScan
.addFamily(HConstants
.REPLICATION_BARRIER_FAMILY
);
3900 .withStartRow(ClientMetaTableAccessor
.getTableStartRowForMeta(cleanReplicationBarrierTable
,
3901 ClientMetaTableAccessor
.QueryType
.REGION
))
3902 .withStopRow(ClientMetaTableAccessor
.getTableStopRowForMeta(cleanReplicationBarrierTable
,
3903 ClientMetaTableAccessor
.QueryType
.REGION
));
3905 try (ResultScanner scanner
= meta
.getScanner(barrierScan
)) {
3906 while ((result
= scanner
.next()) != null) {
3907 regionNames
.add(result
.getRow());
3910 if (regionNames
.size() <= 0) {
3911 errors
.reportError(ERROR_CODE
.INVALID_TABLE
,
3912 "there is no barriers of this table: " + cleanReplicationBarrierTable
);
3915 ReplicationQueueStorage queueStorage
=
3916 ReplicationStorageFactory
.getReplicationQueueStorage(zkw
, getConf());
3917 List
<ReplicationPeerDescription
> peerDescriptions
= admin
.listReplicationPeers();
3918 if (peerDescriptions
!= null && peerDescriptions
.size() > 0) {
3919 List
<String
> peers
= peerDescriptions
.stream()
3920 .filter(peerConfig
-> peerConfig
.getPeerConfig()
3921 .needToReplicate(cleanReplicationBarrierTable
))
3922 .map(peerConfig
-> peerConfig
.getPeerId()).collect(Collectors
.toList());
3924 List
<String
> batch
= new ArrayList
<>();
3925 for (String peer
: peers
) {
3926 for (byte[] regionName
: regionNames
) {
3927 batch
.add(RegionInfo
.encodeRegionName(regionName
));
3928 if (batch
.size() % 100 == 0) {
3929 queueStorage
.removeLastSequenceIds(peer
, batch
);
3933 if (batch
.size() > 0) {
3934 queueStorage
.removeLastSequenceIds(peer
, batch
);
3938 } catch (ReplicationException re
) {
3939 throw new IOException(re
);
3942 for (byte[] regionName
: regionNames
) {
3943 meta
.delete(new Delete(regionName
).addFamily(HConstants
.REPLICATION_BARRIER_FAMILY
));
3949 * ls -r for debugging purposes
3951 void debugLsr(Path p
) throws IOException
{
3952 debugLsr(getConf(), p
, errors
);
3956 * ls -r for debugging purposes
3958 public static void debugLsr(Configuration conf
,
3959 Path p
) throws IOException
{
3960 debugLsr(conf
, p
, new PrintingErrorReporter());
3964 * ls -r for debugging purposes
3966 public static void debugLsr(Configuration conf
,
3967 Path p
, HbckErrorReporter errors
) throws IOException
{
3968 if (!LOG
.isDebugEnabled() || p
== null) {
3971 FileSystem fs
= p
.getFileSystem(conf
);
3973 if (!fs
.exists(p
)) {
3977 errors
.print(p
.toString());
3983 if (fs
.getFileStatus(p
).isDirectory()) {
3984 FileStatus
[] fss
= fs
.listStatus(p
);
3985 for (FileStatus status
: fss
) {
3986 debugLsr(conf
, status
.getPath(), errors
);