Class HBaseFsck
- All Implemented Interfaces:
Closeable
,AutoCloseable
,org.apache.hadoop.conf.Configurable
Region consistency checks verify that hbase:meta, region deployment on region servers and the state of data in HDFS (.regioninfo files) all are in accordance.
Table integrity checks verify that all possible row keys resolve to exactly one region of a table. This means there are no individual degenerate or backwards regions; no holes between regions; and that there are no overlapping regions.
The general repair strategy works in two phases:
- Repair Table Integrity on HDFS. (merge or fabricate regions)
- Repair Region Consistency with hbase:meta and assignments
For table integrity repairs, the tables' region directories are scanned for .regioninfo files. Each table's integrity is then verified. If there are any orphan regions (regions with no .regioninfo files) or holes, new regions are fabricated. Backwards regions are sidelined as well as empty degenerate (endkey==startkey) regions. If there are any overlapping regions, a new region is created and all data is merged into the new region.
Table integrity repairs deal solely with HDFS and could potentially be done offline -- the hbase region servers or master do not need to be running. This phase can eventually be used to completely reconstruct the hbase:meta table in an offline fashion.
Region consistency requires three conditions -- 1) valid .regioninfo file present in an HDFS region dir, 2) valid row with .regioninfo data in META, and 3) a region is deployed only at the regionserver that was assigned to with proper state in the master.
Region consistency repairs require hbase to be online so that hbck can contact the HBase master and region servers. The hbck#connect() method must first be called successfully. Much of the region consistency information is transient and less risky to repair.
If hbck is run from the command line, there are a handful of arguments that can be used to limit
the kinds of repairs hbck will do. See the code in printUsageAndExit()
for more details.
-
Nested Class Summary
Modifier and TypeClassDescription(package private) class
Deprecated.private static class
Deprecated.(package private) static class
Deprecated.This is a Tool wrapper that gathers -Dxxx=yyy configuration settings from the command line.(package private) static class
Deprecated.private static class
Deprecated.(package private) class
Deprecated.Contact hdfs and get all information about specified table directory into regioninfo list.(package private) static class
Deprecated.Contact hdfs and get all information about specified table directory into regioninfo list.(package private) static class
Deprecated.(package private) static class
Deprecated.Contact a region server and get all information from it -
Field Summary
Modifier and TypeFieldDescriptionprivate Admin
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private TableName
Deprecated.private Connection
Deprecated.private final RetryCounterFactory
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.private static final int
Deprecated.static final long
Deprecated.static final long
Deprecated.private static final int
Deprecated.private static boolean
Deprecated.OptionsDeprecated.private final HbckErrorReporter
Deprecated.Stateprotected ExecutorService
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.(package private) int
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private boolean
Deprecated.private static boolean
Deprecated.static final String
Deprecated.Here is where hbase-1.x used to default the lock for hbck1.private org.apache.hadoop.fs.Path
Deprecated.private String
Deprecated.private final AtomicBoolean
Deprecated.private org.apache.hadoop.fs.FSDataOutputStream
Deprecated.private boolean
Deprecated.private HFileCorruptionChecker
Deprecated.private boolean
Deprecated.private final RetryCounterFactory
Deprecated.private static final org.slf4j.Logger
Deprecated.Internal resourcesprivate static final int
Deprecated.private int
Deprecated.private int
Deprecated.private Table
Deprecated.private List<HbckRegionInfo>
Deprecated.When initially looking at HDFS, we attempt to find any orphaned data.Deprecated.private TreeMap<String,
HbckRegionInfo> Deprecated.This map contains the state of all hbck items.private boolean
Deprecated.private boolean
Deprecated.private int
Deprecated.private static boolean
Deprecated.private boolean
Deprecated.private org.apache.hadoop.fs.Path
Deprecated.Deprecated.private long
Deprecated.private ClusterMetrics
Deprecated.private static boolean
Deprecated.Deprecated.private SortedMap<TableName,
HbckTableInfo> Deprecated.This map from Tablename -> TableInfo contains the structures necessary to detect table consistency problems (holes, dupes, overlaps).private Map<TableName,
TableState> Deprecated.private long
Deprecated.Deprecated.private ZKWatcher
Deprecated. -
Constructor Summary
ConstructorDescriptionHBaseFsck
(org.apache.hadoop.conf.Configuration conf) Deprecated.ConstructorHBaseFsck
(org.apache.hadoop.conf.Configuration conf, ExecutorService exec) Deprecated.Constructor Configuration object if the master is not running if unable to connect to ZooKeeper -
Method Summary
Modifier and TypeMethodDescriptionprivate void
Deprecated.private void
Deprecated.Orphaned regions are regions without a .regioninfo file in them.private void
adoptHdfsOrphans
(Collection<HbckRegionInfo> orphanHdfsDirs) Deprecated.Iterates through the list of all orphan/invalid regiondirs.private void
assignMetaReplica
(int replicaId) Deprecated.private void
Deprecated.Check consistency of all regions that have been found in previous phases.private void
Deprecated.private void
Deprecated.Check and fix table states, assumes full info available: - tableInfos - empty tables loadedstatic Pair<org.apache.hadoop.fs.Path,
org.apache.hadoop.fs.FSDataOutputStream> checkAndMarkRunningHbck
(org.apache.hadoop.conf.Configuration conf, RetryCounter retryCounter) Deprecated.This method maintains a lock using a file.private SortedMap<TableName,
HbckTableInfo> checkHdfsIntegrity
(boolean fixHoles, boolean fixOverlaps) Deprecated.(package private) SortedMap<TableName,
HbckTableInfo> Deprecated.Checks tables integrity.(package private) boolean
Deprecated.Check values in regionInfo for hbase:meta Check if zero or more than one regions with hbase:meta are found.void
Deprecated.private void
checkRegionConsistency
(String key, HbckRegionInfo hbi) Deprecated.Check a single region for consistency and correct deployment.private void
Deprecated.Check consistency of all regions using multiple threads concurrently.void
Deprecated.private void
Deprecated.private void
Deprecated.Clear the current state of hbck.void
close()
Deprecated.(package private) void
Deprecated.Attempts to undeploy a region from a region server based in information in META.void
connect()
Deprecated.To repair region consistency, one must call connect() in order to repair online state.protected HFileCorruptionChecker
createHFileCorruptionChecker
(boolean sidelineCorruptHFiles) Deprecated.static RetryCounterFactory
createLockRetryCounterFactory
(org.apache.hadoop.conf.Configuration conf) Deprecated.Returns A retry counter factory configured for retrying lock file creation.private static ExecutorService
createThreadPool
(org.apache.hadoop.conf.Configuration conf) Deprecated.private static RetryCounterFactory
createZnodeRetryCounterFactory
(org.apache.hadoop.conf.Configuration conf) Deprecated.Returns A retry counter factory configured for retrying znode creation.private ZKWatcher
Deprecated.static void
debugLsr
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path p) Deprecated.ls -r for debugging purposesstatic void
debugLsr
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path p, HbckErrorReporter errors) Deprecated.ls -r for debugging purposes(package private) void
debugLsr
(org.apache.hadoop.fs.Path p) Deprecated.ls -r for debugging purposesprivate void
deleteMetaRegion
(byte[] metaKey) Deprecated.Deletes region from meta tableprivate void
Deprecated.Deletes region from meta tableexec
(ExecutorService exec, String[] args) Deprecated.private boolean
fabricateTableInfo
(FSTableDescriptors fstd, TableName tableName, Set<String> columns) Deprecated.To fabricate a .tableinfo file with following contents
1.void
Deprecated.To fix the empty REGIONINFO_QUALIFIER rows from hbase:metavoid
Deprecated.To fix orphan table by creating a .tableinfo file under tableDir
1.getColumnFamilyList
(Set<String> columns, HbckRegionInfo hbi) Deprecated.To get the column family list according to the column family dirs(package private) static HbckErrorReporter
getErrorReporter
(org.apache.hadoop.conf.Configuration conf) Deprecated.Deprecated.Deprecated.Deprecated.int
Deprecated.int
Deprecated.private HbckRegionInfo
getOrCreateInfo
(String name) Deprecated.Gets the entry in regionInfo corresponding to the the given encoded region name.int
Deprecated.(package private) org.apache.hadoop.fs.Path
Deprecated.(package private) TableDescriptor[]
getTableDescriptors
(List<TableName> tableNames) Deprecated.(package private) TableDescriptor[]
getTables
(AtomicInteger numSkipped) Deprecated.Return a list of user-space table names whose metadata have not been modified in the last few milliseconds specified by timelag if any of the REGIONINFO_QUALIFIER, SERVER_QUALIFIER, STARTCODE_QUALIFIER, SPLITA_QUALIFIER, SPLITB_QUALIFIER have not changed in the last milliseconds specified by timelag, then the table is a candidate to be returned.static org.apache.hadoop.fs.Path
getTmpDir
(org.apache.hadoop.conf.Configuration conf) Deprecated.Returns Return the tmp dir this tool writes too.void
includeTable
(TableName table) Deprecated.boolean
Deprecated.Only one instance of hbck can modify HBase at a time.private boolean
isOptionsSupported
(String[] args) Deprecated.(package private) boolean
isTableDisabled
(TableName tableName) Deprecated.Check if the specified region's table is disabled.(package private) boolean
isTableIncluded
(TableName table) Deprecated.Only check/fix tables specified by the list, Empty list means all tables are included.static byte[]
keyOnly
(byte[] b) Deprecated.private void
Deprecated.Get deployed regions according to the region servers.void
Deprecated.Scan HDFS for all regions, recording their information into regionInfoMapprivate SortedMap<TableName,
HbckTableInfo> Deprecated.Populate hbi's from regionInfos loaded from file system.(package private) boolean
Deprecated.Scan hbase:meta, adding all regions found to the regionInfo map.private void
Deprecated.Loads table info's for tables that may not have been included, since there are no regions reported for the table, but table dir is there in hdfsprivate void
Deprecated.Load the list of disabled tables in ZK into local set.private void
Deprecated.Log an appropriate message about whether or not overlapping merges are computed in parallel.static void
Deprecated.Main programint
mergeRegionDirs
(org.apache.hadoop.fs.Path targetRegionDir, HbckRegionInfo contained) Deprecated.Merge hdfs data by moving from contained HbckRegionInfo into targetRegionDir.(package private) void
offline
(byte[] regionName) Deprecated.This backwards-compatibility wrapper for permanently offlining a region that should not be alive.void
Deprecated.This repair method analyzes hbase data in hdfs and repairs it to satisfy the table integrity rules.private void
Deprecated.Scan all the store file names to find any lingering HFileLink files, which refer to some none-exiting files.private void
Deprecated.Scan all the store file names to find any lingering reference files, which refer to some none-exiting files.int
Deprecated.This repair method requires the cluster to be online since it contacts region servers and the masters.int
Deprecated.Contacts the master and prints out cluster-wide informationprivate void
Deprecated.private void
printTableSummary
(SortedMap<TableName, HbckTableInfo> tablesInfo) Deprecated.Prints summary of all tables found on the system.protected HBaseFsck
Deprecated.(package private) void
processRegionServers
(Collection<ServerName> regionServerList) Deprecated.Contacts each regionserver and fetches metadata about regions.private boolean
Deprecated.Record the location of the hbase:meta region as found in ZooKeeper.private void
Deprecated.TODO -- need to add tests for this.private void
Deprecated.TODO -- need to add tests for this.private void
Deprecated.private void
Deprecated.Reset the split parent region info in meta tableprivate int
Deprecated.This method determines if there are table integrity errors in HDFS.void
setCheckHdfs
(boolean checking) Deprecated.(package private) void
Deprecated.Set hbase:meta check mode.void
setCleanReplicationBarrier
(boolean shouldClean) Deprecated.void
setCleanReplicationBarrierTable
(String cleanReplicationBarrierTable) Deprecated.static void
Deprecated.Display the full report from fsck.void
setFixAssignments
(boolean shouldFix) Deprecated.Fix inconsistencies found by fsck.void
setFixEmptyMetaCells
(boolean shouldFix) Deprecated.void
setFixHdfsHoles
(boolean shouldFix) Deprecated.void
setFixHdfsOrphans
(boolean shouldFix) Deprecated.void
setFixHdfsOverlaps
(boolean shouldFix) Deprecated.void
setFixHFileLinks
(boolean shouldFix) Deprecated.void
setFixMeta
(boolean shouldFix) Deprecated.void
setFixReferenceFiles
(boolean shouldFix) Deprecated.void
setFixReplication
(boolean shouldFix) Deprecated.Set replication fix mode.void
setFixSplitParents
(boolean shouldFix) Deprecated.void
setFixTableOrphans
(boolean shouldFix) Deprecated.void
setFixVersionFile
(boolean shouldFix) Deprecated.static void
Deprecated.Set exclusive mode.void
Deprecated.void
setIgnorePreCheckPermission
(boolean ignorePreCheckPermission) Deprecated.private boolean
Deprecated.This method maintains an ephemeral znode.void
setMaxMerge
(int mm) Deprecated.void
setMaxOverlapsToSideline
(int mo) Deprecated.(package private) void
Deprecated.Set region boundaries check mode.void
setRemoveParents
(boolean shouldFix) Deprecated.void
setRetCode
(int code) Deprecated.(package private) void
Deprecated.Check if we should rerun fsck again.void
setSidelineBigOverlaps
(boolean sbo) Deprecated.void
setSidelineDir
(String sidelineDir) Deprecated.(package private) static void
Deprecated.Set summary mode.void
setTimeLag
(long seconds) Deprecated.We are interested in only those tables that have not changed their state in hbase:meta during the last few seconds specified by hbase.admin.fsck.timelag(package private) boolean
Deprecated.static boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.(package private) boolean
Deprecated.boolean
Deprecated.boolean
Deprecated.(package private) boolean
Deprecated.boolean
Deprecated.boolean
Deprecated.private boolean
sidelineFile
(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path hbaseRoot, org.apache.hadoop.fs.Path path) Deprecated.(package private) org.apache.hadoop.fs.Path
sidelineRegionDir
(org.apache.hadoop.fs.FileSystem fs, String parentDir, HbckRegionInfo hi) Deprecated.Sideline a region dir (instead of deleting it)(package private) org.apache.hadoop.fs.Path
sidelineRegionDir
(org.apache.hadoop.fs.FileSystem fs, HbckRegionInfo hi) Deprecated.Sideline a region dir (instead of deleting it)private void
tryAssignmentRepair
(HbckRegionInfo hbi, String msg) Deprecated.private void
Deprecated.private void
Deprecated.private void
Deprecated.private void
Deprecated.Methods inherited from class org.apache.hadoop.conf.Configured
getConf, setConf
-
Field Details
-
DEFAULT_TIME_LAG
Deprecated.- See Also:
-
DEFAULT_SLEEP_BEFORE_RERUN
Deprecated.- See Also:
-
MAX_NUM_THREADS
Deprecated.- See Also:
-
rsSupportsOffline
Deprecated. -
DEFAULT_OVERLAPS_TO_SIDELINE
Deprecated.- See Also:
-
DEFAULT_MAX_MERGE
Deprecated.- See Also:
-
HBCK_LOCK_FILE
Deprecated.Here is where hbase-1.x used to default the lock for hbck1. It puts in place a lock when it goes to write/make changes.- See Also:
-
DEFAULT_MAX_LOCK_FILE_ATTEMPTS
Deprecated.- See Also:
-
DEFAULT_LOCK_FILE_ATTEMPT_SLEEP_INTERVAL
Deprecated.- See Also:
-
DEFAULT_LOCK_FILE_ATTEMPT_MAX_SLEEP_TIME
Deprecated.- See Also:
-
DEFAULT_WAIT_FOR_LOCK_TIMEOUT
Deprecated.- See Also:
-
DEFAULT_MAX_CREATE_ZNODE_ATTEMPTS
Deprecated.- See Also:
-
DEFAULT_CREATE_ZNODE_ATTEMPT_SLEEP_INTERVAL
Deprecated.- See Also:
-
DEFAULT_CREATE_ZNODE_ATTEMPT_MAX_SLEEP_TIME
Deprecated.- See Also:
-
LOG
Deprecated.Internal resources -
status
Deprecated. -
connection
Deprecated. -
admin
Deprecated. -
meta
Deprecated. -
executor
Deprecated. -
startMillis
Deprecated. -
hfcc
Deprecated. -
retcode
Deprecated. -
HBCK_LOCK_PATH
Deprecated. -
hbckOutFd
Deprecated. -
hbckLockCleanup
Deprecated. -
unsupportedOptionsInV2
Deprecated. -
details
Deprecated.Options -
timelag
Deprecated. -
forceExclusive
Deprecated. -
fixAssignments
Deprecated. -
fixMeta
Deprecated. -
checkHdfs
Deprecated. -
fixHdfsHoles
Deprecated. -
fixHdfsOverlaps
Deprecated. -
fixHdfsOrphans
Deprecated. -
fixTableOrphans
Deprecated. -
fixVersionFile
Deprecated. -
fixSplitParents
Deprecated. -
removeParents
Deprecated. -
fixReferenceFiles
Deprecated. -
fixHFileLinks
Deprecated. -
fixEmptyMetaCells
Deprecated. -
fixReplication
Deprecated. -
cleanReplicationBarrier
Deprecated. -
fixAny
Deprecated. -
tablesIncluded
Deprecated. -
cleanReplicationBarrierTable
Deprecated. -
maxMerge
Deprecated. -
maxOverlapsToSideline
Deprecated. -
sidelineBigOverlaps
Deprecated. -
sidelineDir
Deprecated. -
rerun
Deprecated. -
summary
Deprecated. -
checkMetaOnly
Deprecated. -
checkRegionBoundaries
Deprecated. -
ignorePreCheckPermission
Deprecated. -
errors
Deprecated.State -
fixes
int fixesDeprecated. -
regionInfoMap
Deprecated.This map contains the state of all hbck items. It maps from encoded region name to HbckRegionInfo structure. The information contained in HbckRegionInfo is used to detect and correct consistency (hdfs/meta/deployment) problems. -
emptyRegionInfoQualifiers
Deprecated. -
tablesInfo
Deprecated.This map from Tablename -> TableInfo contains the structures necessary to detect table consistency problems (holes, dupes, overlaps). It is sorted to prevent dupes. If tablesIncluded is empty, this map contains all tables. Otherwise, it contains only meta tables and tables in tablesIncluded, unless checkMetaOnly is specified, in which case, it contains only the meta table -
orphanHdfsDirs
Deprecated.When initially looking at HDFS, we attempt to find any orphaned data. -
orphanTableDirs
Deprecated. -
tableStates
Deprecated. -
lockFileRetryCounterFactory
Deprecated. -
createZNodeRetryCounterFactory
Deprecated. -
skippedRegions
Deprecated. -
zkw
Deprecated. -
hbckEphemeralNodePath
Deprecated. -
hbckZodeCreated
Deprecated.
-
-
Constructor Details
-
HBaseFsck
public HBaseFsck(org.apache.hadoop.conf.Configuration conf) throws IOException, ClassNotFoundException Deprecated.Constructor- Parameters:
conf
- Configuration object- Throws:
MasterNotRunningException
- if the master is not runningZooKeeperConnectionException
- if unable to connect to ZooKeeperIOException
ClassNotFoundException
-
HBaseFsck
public HBaseFsck(org.apache.hadoop.conf.Configuration conf, ExecutorService exec) throws MasterNotRunningException, ZooKeeperConnectionException, IOException, ClassNotFoundException Deprecated.Constructor Configuration object if the master is not running if unable to connect to ZooKeeper
-
-
Method Details
-
createThreadPool
Deprecated. -
createLockRetryCounterFactory
public static RetryCounterFactory createLockRetryCounterFactory(org.apache.hadoop.conf.Configuration conf) Deprecated.Returns A retry counter factory configured for retrying lock file creation. -
createZnodeRetryCounterFactory
private static RetryCounterFactory createZnodeRetryCounterFactory(org.apache.hadoop.conf.Configuration conf) Deprecated.Returns A retry counter factory configured for retrying znode creation. -
getTmpDir
@Private public static org.apache.hadoop.fs.Path getTmpDir(org.apache.hadoop.conf.Configuration conf) throws IOException Deprecated.Returns Return the tmp dir this tool writes too.- Throws:
IOException
-
checkAndMarkRunningHbck
public static Pair<org.apache.hadoop.fs.Path,org.apache.hadoop.fs.FSDataOutputStream> checkAndMarkRunningHbck(org.apache.hadoop.conf.Configuration conf, RetryCounter retryCounter) throws IOException Deprecated.This method maintains a lock using a file. If the creation fails we return null- Returns:
- FSDataOutputStream object corresponding to the newly opened lock file
- Throws:
IOException
- if IO failure occurs
-
unlockHbck
Deprecated. -
connect
Deprecated.To repair region consistency, one must call connect() in order to repair online state.- Throws:
IOException
-
loadDeployedRegions
Deprecated.Get deployed regions according to the region servers.- Throws:
IOException
InterruptedException
-
clearState
Deprecated.Clear the current state of hbck. -
offlineHdfsIntegrityRepair
Deprecated.This repair method analyzes hbase data in hdfs and repairs it to satisfy the table integrity rules. HBase doesn't need to be online for this operation to work.- Throws:
IOException
InterruptedException
-
onlineConsistencyRepair
public int onlineConsistencyRepair() throws IOException, org.apache.zookeeper.KeeperException, InterruptedExceptionDeprecated.This repair method requires the cluster to be online since it contacts region servers and the masters. It makes each region's state in HDFS, in hbase:meta, and deployments consistent.- Returns:
- If > 0 , number of errors detected, if < 0 there was an unrecoverable error. If 0, we have a clean hbase.
- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
reportUnknownServers
Deprecated.- Throws:
IOException
-
setMasterInMaintenanceMode
Deprecated.This method maintains an ephemeral znode. If the creation fails we return false or throw exception- Returns:
- true if creating znode succeeds; false otherwise
- Throws:
IOException
- if IO failure occurs
-
cleanupHbckZnode
Deprecated. -
onlineHbck
public int onlineHbck() throws IOException, org.apache.zookeeper.KeeperException, InterruptedException, ReplicationExceptionDeprecated.Contacts the master and prints out cluster-wide information- Returns:
- 0 on success, non-zero on failure
- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
ReplicationException
-
keyOnly
Deprecated. -
close
Deprecated.- Specified by:
close
in interfaceAutoCloseable
- Specified by:
close
in interfaceCloseable
- Throws:
IOException
-
checkRegionBoundaries
Deprecated. -
adoptHdfsOrphans
Deprecated.Iterates through the list of all orphan/invalid regiondirs.- Throws:
IOException
-
adoptHdfsOrphan
Deprecated.Orphaned regions are regions without a .regioninfo file in them. We "adopt" these orphans by creating a new region, and moving the column families, recovered edits, WALs, into the new region dir. We determine the region startkey and endkeys by looking at all of the hfiles inside the column families to identify the min and max keys. The resulting region will likely violate table integrity but will be dealt with by merging overlapping regions.- Throws:
IOException
-
restoreHdfsIntegrity
Deprecated.This method determines if there are table integrity errors in HDFS. If there are errors and the appropriate "fix" options are enabled, the method will first correct orphan regions making them into legit regiondirs, and then reload to merge potentially overlapping regions.- Returns:
- number of table integrity errors found
- Throws:
IOException
InterruptedException
-
offlineReferenceFileRepair
Deprecated.Scan all the store file names to find any lingering reference files, which refer to some none-exiting files. If "fix" option is enabled, any lingering reference file will be sidelined if found.Lingering reference file prevents a region from opening. It has to be fixed before a cluster can start properly.
- Throws:
IOException
InterruptedException
-
offlineHLinkFileRepair
Deprecated.Scan all the store file names to find any lingering HFileLink files, which refer to some none-exiting files. If "fix" option is enabled, any lingering HFileLink file will be sidelined if found.- Throws:
IOException
InterruptedException
-
sidelineFile
private boolean sidelineFile(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path hbaseRoot, org.apache.hadoop.fs.Path path) throws IOException Deprecated.- Throws:
IOException
-
reportEmptyMetaCells
Deprecated.TODO -- need to add tests for this. -
reportTablesInFlux
Deprecated.TODO -- need to add tests for this. -
getErrors
Deprecated. -
loadHdfsRegionInfos
private SortedMap<TableName,HbckTableInfo> loadHdfsRegionInfos() throws IOException, InterruptedExceptionDeprecated.Populate hbi's from regionInfos loaded from file system.- Throws:
IOException
InterruptedException
-
getColumnFamilyList
Deprecated.To get the column family list according to the column family dirs- Returns:
- a set of column families
- Throws:
IOException
-
fabricateTableInfo
private boolean fabricateTableInfo(FSTableDescriptors fstd, TableName tableName, Set<String> columns) throws IOException Deprecated.To fabricate a .tableinfo file with following contents
1. the correct tablename
2. the correct colfamily list
3. the default properties for bothTableDescriptor
andColumnFamilyDescriptor
- Throws:
IOException
-
fixEmptyMetaCells
Deprecated.To fix the empty REGIONINFO_QUALIFIER rows from hbase:meta- Throws:
IOException
-
fixOrphanTables
Deprecated.To fix orphan table by creating a .tableinfo file under tableDir
1. if TableInfo is cached, to recover the .tableinfo accordingly
2. else create a default .tableinfo file with following items
2.1 the correct tablename
2.2 the correct colfamily list
2.3 the default properties for bothTableDescriptor
andColumnFamilyDescriptor
- Throws:
IOException
-
logParallelMerge
Deprecated.Log an appropriate message about whether or not overlapping merges are computed in parallel. -
checkHdfsIntegrity
private SortedMap<TableName,HbckTableInfo> checkHdfsIntegrity(boolean fixHoles, boolean fixOverlaps) throws IOException Deprecated.- Throws:
IOException
-
getSidelineDir
Deprecated.- Throws:
IOException
-
sidelineRegionDir
org.apache.hadoop.fs.Path sidelineRegionDir(org.apache.hadoop.fs.FileSystem fs, HbckRegionInfo hi) throws IOException Deprecated.Sideline a region dir (instead of deleting it)- Throws:
IOException
-
sidelineRegionDir
org.apache.hadoop.fs.Path sidelineRegionDir(org.apache.hadoop.fs.FileSystem fs, String parentDir, HbckRegionInfo hi) throws IOException Deprecated.Sideline a region dir (instead of deleting it)- Parameters:
parentDir
- if specified, the region will be sidelined to folder like .../parentDir/<table name>/<region name>. The purpose is to group together similar regions sidelined, for example, those regions should be bulk loaded back later on. If NULL, it is ignored.- Throws:
IOException
-
loadTableStates
Deprecated.Load the list of disabled tables in ZK into local set.- Throws:
IOException
-
isTableDisabled
Deprecated.Check if the specified region's table is disabled.- Parameters:
tableName
- table to check status of
-
loadHdfsRegionDirs
Deprecated.Scan HDFS for all regions, recording their information into regionInfoMap- Throws:
IOException
InterruptedException
-
recordMetaRegion
Deprecated.Record the location of the hbase:meta region as found in ZooKeeper.- Throws:
IOException
-
createZooKeeperWatcher
Deprecated.- Throws:
IOException
-
processRegionServers
void processRegionServers(Collection<ServerName> regionServerList) throws IOException, InterruptedException Deprecated.Contacts each regionserver and fetches metadata about regions.- Parameters:
regionServerList
- - the list of region servers to connect to- Throws:
IOException
- if a remote or network exception occursInterruptedException
-
checkAndFixConsistency
private void checkAndFixConsistency() throws IOException, org.apache.zookeeper.KeeperException, InterruptedExceptionDeprecated.Check consistency of all regions that have been found in previous phases.- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
checkRegionConsistencyConcurrently
private void checkRegionConsistencyConcurrently(List<HBaseFsck.CheckRegionConsistencyWorkItem> workItems) throws IOException, org.apache.zookeeper.KeeperException, InterruptedException Deprecated.Check consistency of all regions using multiple threads concurrently.- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
addSkippedRegion
Deprecated. -
checkAndFixTableStates
Deprecated.Check and fix table states, assumes full info available: - tableInfos - empty tables loaded- Throws:
IOException
-
preCheckPermission
Deprecated.- Throws:
IOException
-
deleteMetaRegion
Deprecated.Deletes region from meta table- Throws:
IOException
-
deleteMetaRegion
Deprecated.Deletes region from meta table- Throws:
IOException
-
resetSplitParent
Deprecated.Reset the split parent region info in meta table- Throws:
IOException
-
offline
Deprecated.This backwards-compatibility wrapper for permanently offlining a region that should not be alive. If the region server does not support the "offline" method, it will use the closest unassign method instead. This will basically work until one attempts to disable or delete the affected table. The problem has to do with in-memory only master state, so restarting the HMaster or failing over to another should fix this.- Throws:
IOException
-
closeRegion
Deprecated.Attempts to undeploy a region from a region server based in information in META. Any operations that modify the file system should make sure that its corresponding region is not deployed to prevent data races. A separate call is required to update the master in-memory region state kept in the AssignementManager. Because disable uses this state instead of that found in META, we can't seem to cleanly disable/delete tables that have been hbck fixed. When used on a version of HBase that does not have the offline ipc call exposed on the master (<0.90.5, <0.92.0) a master restart or failover may be required.- Throws:
IOException
InterruptedException
-
undeployRegions
Deprecated.- Throws:
IOException
InterruptedException
-
undeployRegionsForHbi
Deprecated.- Throws:
IOException
InterruptedException
-
tryAssignmentRepair
private void tryAssignmentRepair(HbckRegionInfo hbi, String msg) throws IOException, org.apache.zookeeper.KeeperException, InterruptedException Deprecated.- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
checkRegionConsistency
private void checkRegionConsistency(String key, HbckRegionInfo hbi) throws IOException, org.apache.zookeeper.KeeperException, InterruptedException Deprecated.Check a single region for consistency and correct deployment.- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
checkIntegrity
Deprecated.Checks tables integrity. Goes over all regions and scans the tables. Collects all the pieces for each table and checks if there are missing, repeated or overlapping ones.- Throws:
IOException
-
loadTableInfosForTablesWithNoRegion
Deprecated.Loads table info's for tables that may not have been included, since there are no regions reported for the table, but table dir is there in hdfs- Throws:
IOException
-
mergeRegionDirs
public int mergeRegionDirs(org.apache.hadoop.fs.Path targetRegionDir, HbckRegionInfo contained) throws IOException Deprecated.Merge hdfs data by moving from contained HbckRegionInfo into targetRegionDir.- Returns:
- number of file move fixes done to merge regions.
- Throws:
IOException
-
getTables
Deprecated.Return a list of user-space table names whose metadata have not been modified in the last few milliseconds specified by timelag if any of the REGIONINFO_QUALIFIER, SERVER_QUALIFIER, STARTCODE_QUALIFIER, SPLITA_QUALIFIER, SPLITB_QUALIFIER have not changed in the last milliseconds specified by timelag, then the table is a candidate to be returned.- Returns:
- tables that have not been modified recently
- Throws:
IOException
- if an error is encountered
-
getTableDescriptors
Deprecated. -
getOrCreateInfo
Deprecated.Gets the entry in regionInfo corresponding to the the given encoded region name. If the region has not been seen yet, a new entry is added and returned. -
checkAndFixReplication
Deprecated.- Throws:
ReplicationException
IOException
-
checkMetaRegion
boolean checkMetaRegion() throws IOException, org.apache.zookeeper.KeeperException, InterruptedExceptionDeprecated.Check values in regionInfo for hbase:meta Check if zero or more than one regions with hbase:meta are found. If there are inconsistencies (i.e. zero or more than one regions pretend to be holding the hbase:meta) try to fix that and report an error.- Throws:
IOException
- from HBaseFsckRepair functionsorg.apache.zookeeper.KeeperException
InterruptedException
-
unassignMetaReplica
private void unassignMetaReplica(HbckRegionInfo hi) throws IOException, InterruptedException, org.apache.zookeeper.KeeperException Deprecated.- Throws:
IOException
InterruptedException
org.apache.zookeeper.KeeperException
-
assignMetaReplica
private void assignMetaReplica(int replicaId) throws IOException, org.apache.zookeeper.KeeperException, InterruptedException Deprecated.- Throws:
IOException
org.apache.zookeeper.KeeperException
InterruptedException
-
loadMetaEntries
Deprecated.Scan hbase:meta, adding all regions found to the regionInfo map.- Throws:
IOException
- if an error is encountered
-
printTableSummary
Deprecated.Prints summary of all tables found on the system. -
getErrorReporter
static HbckErrorReporter getErrorReporter(org.apache.hadoop.conf.Configuration conf) throws ClassNotFoundException Deprecated.- Throws:
ClassNotFoundException
-
setDisplayFullReport
Deprecated.Display the full report from fsck. This displays all live and dead region servers, and all known regions. -
shouldDisplayFullReport
Deprecated. -
setForceExclusive
Deprecated.Set exclusive mode. -
isExclusive
Deprecated.Only one instance of hbck can modify HBase at a time. -
setSummary
Deprecated.Set summary mode. Print only summary of the tables and status (OK or INCONSISTENT) -
setCheckMetaOnly
void setCheckMetaOnly()Deprecated.Set hbase:meta check mode. Print only info about hbase:meta table deployment/state -
setRegionBoundariesCheck
void setRegionBoundariesCheck()Deprecated.Set region boundaries check mode. -
setFixReplication
Deprecated.Set replication fix mode. -
setCleanReplicationBarrier
Deprecated. -
setShouldRerun
void setShouldRerun()Deprecated.Check if we should rerun fsck again. This checks if we've tried to fix something and we should rerun fsck tool again. Display the full report from fsck. This displays all live and dead region servers, and all known regions. -
shouldRerun
Deprecated. -
setFixAssignments
Deprecated.Fix inconsistencies found by fsck. This should try to fix errors (if any) found by fsck utility. -
shouldFixAssignments
boolean shouldFixAssignments()Deprecated. -
setFixMeta
Deprecated. -
shouldFixMeta
boolean shouldFixMeta()Deprecated. -
setFixEmptyMetaCells
Deprecated. -
shouldFixEmptyMetaCells
boolean shouldFixEmptyMetaCells()Deprecated. -
setCheckHdfs
Deprecated. -
shouldCheckHdfs
boolean shouldCheckHdfs()Deprecated. -
setFixHdfsHoles
Deprecated. -
shouldFixHdfsHoles
boolean shouldFixHdfsHoles()Deprecated. -
setFixTableOrphans
Deprecated. -
shouldFixTableOrphans
boolean shouldFixTableOrphans()Deprecated. -
setFixHdfsOverlaps
Deprecated. -
shouldFixHdfsOverlaps
boolean shouldFixHdfsOverlaps()Deprecated. -
setFixHdfsOrphans
Deprecated. -
shouldFixHdfsOrphans
boolean shouldFixHdfsOrphans()Deprecated. -
setFixVersionFile
Deprecated. -
shouldFixVersionFile
Deprecated. -
setSidelineBigOverlaps
Deprecated. -
shouldSidelineBigOverlaps
Deprecated. -
setFixSplitParents
Deprecated. -
setRemoveParents
Deprecated. -
shouldFixSplitParents
boolean shouldFixSplitParents()Deprecated. -
shouldRemoveParents
boolean shouldRemoveParents()Deprecated. -
setFixReferenceFiles
Deprecated. -
shouldFixReferenceFiles
boolean shouldFixReferenceFiles()Deprecated. -
setFixHFileLinks
Deprecated. -
shouldFixHFileLinks
boolean shouldFixHFileLinks()Deprecated. -
shouldIgnorePreCheckPermission
Deprecated. -
setIgnorePreCheckPermission
Deprecated. -
setMaxMerge
Deprecated.- Parameters:
mm
- maximum number of regions to merge into a single region.
-
getMaxMerge
Deprecated. -
setMaxOverlapsToSideline
Deprecated. -
getMaxOverlapsToSideline
Deprecated. -
isTableIncluded
Deprecated.Only check/fix tables specified by the list, Empty list means all tables are included. -
includeTable
Deprecated. -
getIncludedTables
Deprecated. -
setTimeLag
Deprecated.We are interested in only those tables that have not changed their state in hbase:meta during the last few seconds specified by hbase.admin.fsck.timelag- Parameters:
seconds
- - the time in seconds
-
setSidelineDir
Deprecated.- Parameters:
sidelineDir
- - HDFS path to sideline data
-
createHFileCorruptionChecker
protected HFileCorruptionChecker createHFileCorruptionChecker(boolean sidelineCorruptHFiles) throws IOException Deprecated.- Throws:
IOException
-
getHFilecorruptionChecker
Deprecated. -
setHFileCorruptionChecker
Deprecated. -
setRetCode
Deprecated. -
getRetCode
Deprecated. -
printUsageAndExit
Deprecated. -
main
Deprecated.Main program- Throws:
Exception
-
exec
public HBaseFsck exec(ExecutorService exec, String[] args) throws org.apache.zookeeper.KeeperException, IOException, InterruptedException, ReplicationException Deprecated.- Throws:
org.apache.zookeeper.KeeperException
IOException
InterruptedException
ReplicationException
-
isOptionsSupported
Deprecated. -
setCleanReplicationBarrierTable
Deprecated. -
cleanReplicationBarrier
Deprecated.- Throws:
IOException
-
debugLsr
Deprecated.ls -r for debugging purposes- Throws:
IOException
-
debugLsr
public static void debugLsr(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path p) throws IOException Deprecated.ls -r for debugging purposes- Throws:
IOException
-
debugLsr
public static void debugLsr(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path p, HbckErrorReporter errors) throws IOException Deprecated.ls -r for debugging purposes- Throws:
IOException
-