001/** 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.master; 019 020import java.io.IOException; 021import java.util.HashMap; 022import java.util.HashSet; 023import java.util.LinkedList; 024import java.util.List; 025import java.util.Map; 026import java.util.Set; 027import java.util.concurrent.locks.ReentrantReadWriteLock; 028 029import org.apache.hadoop.fs.FileSystem; 030import org.apache.hadoop.fs.Path; 031import org.apache.hadoop.hbase.ScheduledChore; 032import org.apache.hadoop.hbase.ServerName; 033import org.apache.hadoop.hbase.client.RegionInfo; 034import org.apache.hadoop.hbase.client.TableState; 035import org.apache.hadoop.hbase.util.EnvironmentEdgeManager; 036import org.apache.hadoop.hbase.util.FSUtils; 037import org.apache.hadoop.hbase.util.HbckRegionInfo; 038import org.apache.hadoop.hbase.util.Pair; 039import org.apache.yetus.audience.InterfaceAudience; 040import org.apache.yetus.audience.InterfaceStability; 041import org.slf4j.Logger; 042import org.slf4j.LoggerFactory; 043 044/** 045 * Used to do the hbck checking job at master side. 046 */ 047@InterfaceAudience.Private 048@InterfaceStability.Evolving 049public class HbckChore extends ScheduledChore { 050 private static final Logger LOG = LoggerFactory.getLogger(HbckChore.class.getName()); 051 052 private static final String HBCK_CHORE_INTERVAL = "hbase.master.hbck.chore.interval"; 053 private static final int DEFAULT_HBCK_CHORE_INTERVAL = 60 * 60 * 1000; 054 055 private final MasterServices master; 056 057 /** 058 * This map contains the state of all hbck items. It maps from encoded region 059 * name to HbckRegionInfo structure. The information contained in HbckRegionInfo is used 060 * to detect and correct consistency (hdfs/meta/deployment) problems. 061 */ 062 private final Map<String, HbckRegionInfo> regionInfoMap = new HashMap<>(); 063 064 private final Set<String> disabledTableRegions = new HashSet<>(); 065 066 /** 067 * The regions only opened on RegionServers, but no region info in meta. 068 */ 069 private final Map<String, ServerName> orphanRegionsOnRS = new HashMap<>(); 070 /** 071 * The regions have directory on FileSystem, but no region info in meta. 072 */ 073 private final Map<String, Path> orphanRegionsOnFS = new HashMap<>(); 074 /** 075 * The inconsistent regions. There are three case: 076 * case 1. Master thought this region opened, but no regionserver reported it. 077 * case 2. Master thought this region opened on Server1, but regionserver reported Server2 078 * case 3. More than one regionservers reported opened this region 079 */ 080 private final Map<String, Pair<ServerName, List<ServerName>>> inconsistentRegions = 081 new HashMap<>(); 082 083 /** 084 * The "snapshot" is used to save the last round's HBCK checking report. 085 */ 086 private final Map<String, ServerName> orphanRegionsOnRSSnapshot = new HashMap<>(); 087 private final Map<String, Path> orphanRegionsOnFSSnapshot = new HashMap<>(); 088 private final Map<String, Pair<ServerName, List<ServerName>>> inconsistentRegionsSnapshot = 089 new HashMap<>(); 090 091 /** 092 * The "snapshot" may be changed after checking. And this checking report "snapshot" may be 093 * accessed by web ui. Use this rwLock to synchronize. 094 */ 095 ReentrantReadWriteLock rwLock = new ReentrantReadWriteLock(); 096 097 /** 098 * When running, the "snapshot" may be changed when this round's checking finish. 099 */ 100 private volatile boolean running = false; 101 private volatile long checkingStartTimestamp = 0; 102 private volatile long checkingEndTimestamp = 0; 103 104 private boolean disabled = false; 105 106 public HbckChore(MasterServices master) { 107 super("HbckChore-", master, 108 master.getConfiguration().getInt(HBCK_CHORE_INTERVAL, DEFAULT_HBCK_CHORE_INTERVAL)); 109 this.master = master; 110 int interval = 111 master.getConfiguration().getInt(HBCK_CHORE_INTERVAL, DEFAULT_HBCK_CHORE_INTERVAL); 112 if (interval <= 0) { 113 LOG.warn(HBCK_CHORE_INTERVAL + " is <=0 hence disabling hbck chore"); 114 disableChore(); 115 } 116 } 117 118 @Override 119 protected synchronized void chore() { 120 if (isDisabled() || isRunning()) { 121 LOG.warn("hbckChore is either disabled or is already running. Can't run the chore"); 122 return; 123 } 124 running = true; 125 regionInfoMap.clear(); 126 disabledTableRegions.clear(); 127 orphanRegionsOnRS.clear(); 128 orphanRegionsOnFS.clear(); 129 inconsistentRegions.clear(); 130 checkingStartTimestamp = EnvironmentEdgeManager.currentTime(); 131 loadRegionsFromInMemoryState(); 132 loadRegionsFromRSReport(); 133 try { 134 loadRegionsFromFS(); 135 } catch (IOException e) { 136 LOG.warn("Failed to load the regions from filesystem", e); 137 } 138 saveCheckResultToSnapshot(); 139 running = false; 140 } 141 142 // This function does the sanity checks of making sure the chore is not run when it is 143 // disabled or when it's already running. It returns whether the chore was actually run or not. 144 protected boolean runChore() { 145 if (isDisabled() || isRunning()) { 146 if (isDisabled()) { 147 LOG.warn("hbck chore is disabled! Set " + HBCK_CHORE_INTERVAL + " > 0 to enable it."); 148 } else { 149 LOG.warn("hbck chore already running. Can't run till it finishes."); 150 } 151 return false; 152 } 153 chore(); 154 return true; 155 } 156 157 private void disableChore() { 158 this.disabled = true; 159 } 160 161 public boolean isDisabled() { 162 return this.disabled; 163 } 164 165 private void saveCheckResultToSnapshot() { 166 // Need synchronized here, as this "snapshot" may be access by web ui. 167 rwLock.writeLock().lock(); 168 try { 169 orphanRegionsOnRSSnapshot.clear(); 170 orphanRegionsOnRS.entrySet() 171 .forEach(e -> orphanRegionsOnRSSnapshot.put(e.getKey(), e.getValue())); 172 orphanRegionsOnFSSnapshot.clear(); 173 orphanRegionsOnFS.entrySet() 174 .forEach(e -> orphanRegionsOnFSSnapshot.put(e.getKey(), e.getValue())); 175 inconsistentRegionsSnapshot.clear(); 176 inconsistentRegions.entrySet() 177 .forEach(e -> inconsistentRegionsSnapshot.put(e.getKey(), e.getValue())); 178 checkingEndTimestamp = EnvironmentEdgeManager.currentTime(); 179 } finally { 180 rwLock.writeLock().unlock(); 181 } 182 } 183 184 private void loadRegionsFromInMemoryState() { 185 List<RegionState> regionStates = 186 master.getAssignmentManager().getRegionStates().getRegionStates(); 187 for (RegionState regionState : regionStates) { 188 RegionInfo regionInfo = regionState.getRegion(); 189 if (master.getTableStateManager() 190 .isTableState(regionInfo.getTable(), TableState.State.DISABLED)) { 191 disabledTableRegions.add(regionInfo.getEncodedName()); 192 } 193 HbckRegionInfo.MetaEntry metaEntry = 194 new HbckRegionInfo.MetaEntry(regionInfo, regionState.getServerName(), 195 regionState.getStamp()); 196 regionInfoMap.put(regionInfo.getEncodedName(), new HbckRegionInfo(metaEntry)); 197 } 198 LOG.info("Loaded {} regions from in-memory state of AssignmentManager", regionStates.size()); 199 } 200 201 private void loadRegionsFromRSReport() { 202 int numRegions = 0; 203 Map<ServerName, Set<byte[]>> rsReports = master.getAssignmentManager().getRSReports(); 204 for (Map.Entry<ServerName, Set<byte[]>> entry : rsReports.entrySet()) { 205 ServerName serverName = entry.getKey(); 206 for (byte[] regionName : entry.getValue()) { 207 String encodedRegionName = RegionInfo.encodeRegionName(regionName); 208 HbckRegionInfo hri = regionInfoMap.get(encodedRegionName); 209 if (hri == null) { 210 orphanRegionsOnRS.put(encodedRegionName, serverName); 211 continue; 212 } 213 hri.addServer(hri.getMetaEntry(), serverName); 214 } 215 numRegions += entry.getValue().size(); 216 } 217 LOG.info("Loaded {} regions from {} regionservers' reports and found {} orphan regions", 218 numRegions, rsReports.size(), orphanRegionsOnFS.size()); 219 220 for (Map.Entry<String, HbckRegionInfo> entry : regionInfoMap.entrySet()) { 221 String encodedRegionName = entry.getKey(); 222 HbckRegionInfo hri = entry.getValue(); 223 ServerName locationInMeta = hri.getMetaEntry().getRegionServer(); 224 if (hri.getDeployedOn().size() == 0) { 225 // Because the inconsistent regions are not absolutely right, only skip the offline regions 226 // which belong to disabled table. 227 if (disabledTableRegions.contains(encodedRegionName)) { 228 continue; 229 } 230 // Master thought this region opened, but no regionserver reported it. 231 inconsistentRegions.put(encodedRegionName, new Pair<>(locationInMeta, new LinkedList<>())); 232 } else if (hri.getDeployedOn().size() > 1) { 233 // More than one regionserver reported opened this region 234 inconsistentRegions.put(encodedRegionName, new Pair<>(locationInMeta, hri.getDeployedOn())); 235 } else if (!hri.getDeployedOn().get(0).equals(locationInMeta)) { 236 // Master thought this region opened on Server1, but regionserver reported Server2 237 inconsistentRegions.put(encodedRegionName, new Pair<>(locationInMeta, hri.getDeployedOn())); 238 } 239 } 240 } 241 242 private void loadRegionsFromFS() throws IOException { 243 Path rootDir = master.getMasterFileSystem().getRootDir(); 244 FileSystem fs = master.getMasterFileSystem().getFileSystem(); 245 246 int numRegions = 0; 247 List<Path> tableDirs = FSUtils.getTableDirs(fs, rootDir); 248 for (Path tableDir : tableDirs) { 249 List<Path> regionDirs = FSUtils.getRegionDirs(fs, tableDir); 250 for (Path regionDir : regionDirs) { 251 String encodedRegionName = regionDir.getName(); 252 HbckRegionInfo hri = regionInfoMap.get(encodedRegionName); 253 if (hri == null) { 254 orphanRegionsOnFS.put(encodedRegionName, regionDir); 255 continue; 256 } 257 HbckRegionInfo.HdfsEntry hdfsEntry = new HbckRegionInfo.HdfsEntry(regionDir); 258 hri.setHdfsEntry(hdfsEntry); 259 } 260 numRegions += regionDirs.size(); 261 } 262 LOG.info("Loaded {} tables {} regions from filesyetem and found {} orphan regions", 263 tableDirs.size(), numRegions, orphanRegionsOnFS.size()); 264 } 265 266 /** 267 * When running, the HBCK report may be changed later. 268 */ 269 public boolean isRunning() { 270 return running; 271 } 272 273 /** 274 * @return the regions only opened on RegionServers, but no region info in meta. 275 */ 276 public Map<String, ServerName> getOrphanRegionsOnRS() { 277 // Need synchronized here, as this "snapshot" may be changed after checking. 278 rwLock.readLock().lock(); 279 try { 280 return this.orphanRegionsOnRSSnapshot; 281 } finally { 282 rwLock.readLock().unlock(); 283 } 284 } 285 286 /** 287 * @return the regions have directory on FileSystem, but no region info in meta. 288 */ 289 public Map<String, Path> getOrphanRegionsOnFS() { 290 // Need synchronized here, as this "snapshot" may be changed after checking. 291 rwLock.readLock().lock(); 292 try { 293 return this.orphanRegionsOnFSSnapshot; 294 } finally { 295 rwLock.readLock().unlock(); 296 } 297 } 298 299 /** 300 * Found the inconsistent regions. There are three case: 301 * case 1. Master thought this region opened, but no regionserver reported it. 302 * case 2. Master thought this region opened on Server1, but regionserver reported Server2 303 * case 3. More than one regionservers reported opened this region 304 * 305 * @return the map of inconsistent regions. Key is the region name. Value is a pair of location in 306 * meta and the regionservers which reported opened this region. 307 */ 308 public Map<String, Pair<ServerName, List<ServerName>>> getInconsistentRegions() { 309 // Need synchronized here, as this "snapshot" may be changed after checking. 310 rwLock.readLock().lock(); 311 try { 312 return this.inconsistentRegionsSnapshot; 313 } finally { 314 rwLock.readLock().unlock(); 315 } 316 } 317 318 /** 319 * Used for web ui to show when the HBCK checking started. 320 */ 321 public long getCheckingStartTimestamp() { 322 return this.checkingStartTimestamp; 323 } 324 325 /** 326 * Used for web ui to show when the HBCK checking report generated. 327 */ 328 public long getCheckingEndTimestamp() { 329 return this.checkingEndTimestamp; 330 } 331}