HBASE-19811 Fix findbugs and error-prone warnings in hbase-server (branch-2)
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / wal / AbstractFSWALProvider.java
blobd9badfa99874adb08d46cc461f9111b49eb39c98
1 /**
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.wal;
20 import java.io.FileNotFoundException;
21 import java.io.IOException;
22 import java.util.ArrayList;
23 import java.util.Collections;
24 import java.util.List;
25 import java.util.Objects;
26 import java.util.concurrent.atomic.AtomicBoolean;
27 import java.util.regex.Pattern;
29 import org.apache.hadoop.conf.Configuration;
30 import org.apache.hadoop.fs.FSDataInputStream;
31 import org.apache.hadoop.fs.FileSystem;
32 import org.apache.hadoop.fs.Path;
33 import org.apache.hadoop.hbase.HConstants;
34 import org.apache.hadoop.hbase.ServerName;
35 import org.apache.hadoop.hbase.client.RegionInfo;
36 import org.apache.yetus.audience.InterfaceAudience;
37 import org.apache.yetus.audience.InterfaceStability;
38 import org.slf4j.Logger;
39 import org.slf4j.LoggerFactory;
40 import org.apache.hadoop.hbase.regionserver.wal.AbstractFSWAL;
41 import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
42 import org.apache.hadoop.hbase.util.CancelableProgressable;
43 import org.apache.hadoop.hbase.util.FSUtils;
44 import org.apache.hadoop.hbase.util.LeaseNotRecoveredException;
45 import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
47 /**
48 * Base class of a WAL Provider that returns a single thread safe WAL that writes to Hadoop FS. By
49 * default, this implementation picks a directory in Hadoop FS based on a combination of
50 * <ul>
51 * <li>the HBase root directory
52 * <li>HConstants.HREGION_LOGDIR_NAME
53 * <li>the given factory's factoryId (usually identifying the regionserver by host:port)
54 * </ul>
55 * It also uses the providerId to differentiate among files.
57 @InterfaceAudience.Private
58 @InterfaceStability.Evolving
59 public abstract class AbstractFSWALProvider<T extends AbstractFSWAL<?>> implements WALProvider {
61 private static final Logger LOG = LoggerFactory.getLogger(AbstractFSWALProvider.class);
63 /** Separate old log into different dir by regionserver name **/
64 public static final String SEPARATE_OLDLOGDIR = "hbase.separate.oldlogdir.by.regionserver";
65 public static final boolean DEFAULT_SEPARATE_OLDLOGDIR = false;
67 // Only public so classes back in regionserver.wal can access
68 public interface Reader extends WAL.Reader {
69 /**
70 * @param fs File system.
71 * @param path Path.
72 * @param c Configuration.
73 * @param s Input stream that may have been pre-opened by the caller; may be null.
75 void init(FileSystem fs, Path path, Configuration c, FSDataInputStream s) throws IOException;
78 protected volatile T wal;
79 protected WALFactory factory = null;
80 protected Configuration conf = null;
81 protected List<WALActionsListener> listeners = null;
82 protected String providerId = null;
83 protected AtomicBoolean initialized = new AtomicBoolean(false);
84 // for default wal provider, logPrefix won't change
85 protected String logPrefix = null;
87 /**
88 * we synchronized on walCreateLock to prevent wal recreation in different threads
90 private final Object walCreateLock = new Object();
92 /**
93 * @param factory factory that made us, identity used for FS layout. may not be null
94 * @param conf may not be null
95 * @param listeners may be null
96 * @param providerId differentiate between providers from one factory, used for FS layout. may be
97 * null
99 @Override
100 public void init(WALFactory factory, Configuration conf, List<WALActionsListener> listeners,
101 String providerId) throws IOException {
102 if (!initialized.compareAndSet(false, true)) {
103 throw new IllegalStateException("WALProvider.init should only be called once.");
105 this.factory = factory;
106 this.conf = conf;
107 this.listeners = listeners;
108 this.providerId = providerId;
109 // get log prefix
110 StringBuilder sb = new StringBuilder().append(factory.factoryId);
111 if (providerId != null) {
112 if (providerId.startsWith(WAL_FILE_NAME_DELIMITER)) {
113 sb.append(providerId);
114 } else {
115 sb.append(WAL_FILE_NAME_DELIMITER).append(providerId);
118 logPrefix = sb.toString();
119 doInit(conf);
122 @Override
123 public List<WAL> getWALs() {
124 if (wal == null) {
125 return Collections.emptyList();
127 List<WAL> wals = new ArrayList<>(1);
128 wals.add(wal);
129 return wals;
132 @Override
133 public T getWAL(RegionInfo region) throws IOException {
134 T walCopy = wal;
135 if (walCopy == null) {
136 // only lock when need to create wal, and need to lock since
137 // creating hlog on fs is time consuming
138 synchronized (walCreateLock) {
139 walCopy = wal;
140 if (walCopy == null) {
141 walCopy = createWAL();
142 wal = walCopy;
146 return walCopy;
149 protected abstract T createWAL() throws IOException;
151 protected abstract void doInit(Configuration conf) throws IOException;
153 @Override
154 public void shutdown() throws IOException {
155 T log = this.wal;
156 if (log != null) {
157 log.shutdown();
161 @Override
162 public void close() throws IOException {
163 T log = this.wal;
164 if (log != null) {
165 log.close();
170 * iff the given WALFactory is using the DefaultWALProvider for meta and/or non-meta, count the
171 * number of files (rolled and active). if either of them aren't, count 0 for that provider.
173 @Override
174 public long getNumLogFiles() {
175 T log = this.wal;
176 return log == null ? 0 : log.getNumLogFiles();
180 * iff the given WALFactory is using the DefaultWALProvider for meta and/or non-meta, count the
181 * size of files (only rolled). if either of them aren't, count 0 for that provider.
183 @Override
184 public long getLogFileSize() {
185 T log = this.wal;
186 return log == null ? 0 : log.getLogFileSize();
190 * returns the number of rolled WAL files.
192 @VisibleForTesting
193 public static int getNumRolledLogFiles(WAL wal) {
194 return ((AbstractFSWAL<?>) wal).getNumRolledLogFiles();
198 * returns the size of rolled WAL files.
200 @VisibleForTesting
201 public static long getLogFileSize(WAL wal) {
202 return ((AbstractFSWAL<?>) wal).getLogFileSize();
206 * return the current filename from the current wal.
208 @VisibleForTesting
209 public static Path getCurrentFileName(final WAL wal) {
210 return ((AbstractFSWAL<?>) wal).getCurrentFileName();
214 * request a log roll, but don't actually do it.
216 @VisibleForTesting
217 static void requestLogRoll(final WAL wal) {
218 ((AbstractFSWAL<?>) wal).requestLogRoll();
221 // should be package private; more visible for use in AbstractFSWAL
222 public static final String WAL_FILE_NAME_DELIMITER = ".";
223 /** The hbase:meta region's WAL filename extension */
224 @VisibleForTesting
225 public static final String META_WAL_PROVIDER_ID = ".meta";
226 static final String DEFAULT_PROVIDER_ID = "default";
228 // Implementation details that currently leak in tests or elsewhere follow
229 /** File Extension used while splitting an WAL into regions (HBASE-2312) */
230 public static final String SPLITTING_EXT = "-splitting";
233 * It returns the file create timestamp from the file name. For name format see
234 * {@link #validateWALFilename(String)} public until remaining tests move to o.a.h.h.wal
235 * @param wal must not be null
236 * @return the file number that is part of the WAL file name
238 @VisibleForTesting
239 public static long extractFileNumFromWAL(final WAL wal) {
240 final Path walName = ((AbstractFSWAL<?>) wal).getCurrentFileName();
241 if (walName == null) {
242 throw new IllegalArgumentException("The WAL path couldn't be null");
244 final String[] walPathStrs = walName.toString().split("\\" + WAL_FILE_NAME_DELIMITER);
245 return Long.parseLong(walPathStrs[walPathStrs.length - (isMetaFile(walName) ? 2 : 1)]);
249 * Pattern used to validate a WAL file name see {@link #validateWALFilename(String)} for
250 * description.
252 private static final Pattern pattern = Pattern
253 .compile(".*\\.\\d*(" + META_WAL_PROVIDER_ID + ")*");
256 * A WAL file name is of the format: &lt;wal-name&gt;{@link #WAL_FILE_NAME_DELIMITER}
257 * &lt;file-creation-timestamp&gt;[.meta]. provider-name is usually made up of a server-name and a
258 * provider-id
259 * @param filename name of the file to validate
260 * @return <tt>true</tt> if the filename matches an WAL, <tt>false</tt> otherwise
262 public static boolean validateWALFilename(String filename) {
263 return pattern.matcher(filename).matches();
267 * Construct the directory name for all WALs on a given server. Dir names currently look like
268 * this for WALs: <code>hbase//WALs/kalashnikov.att.net,61634,1486865297088</code>.
269 * @param serverName Server name formatted as described in {@link ServerName}
270 * @return the relative WAL directory name, e.g. <code>.logs/1.example.org,60030,12345</code> if
271 * <code>serverName</code> passed is <code>1.example.org,60030,12345</code>
273 public static String getWALDirectoryName(final String serverName) {
274 StringBuilder dirName = new StringBuilder(HConstants.HREGION_LOGDIR_NAME);
275 dirName.append("/");
276 dirName.append(serverName);
277 return dirName.toString();
281 * Construct the directory name for all old WALs on a given server. The default old WALs dir
282 * looks like: <code>hbase/oldWALs</code>. If you config hbase.separate.oldlogdir.by.regionserver
283 * to true, it looks like <code>hbase//oldWALs/kalashnikov.att.net,61634,1486865297088</code>.
284 * @param conf
285 * @param serverName Server name formatted as described in {@link ServerName}
286 * @return the relative WAL directory name
288 public static String getWALArchiveDirectoryName(Configuration conf, final String serverName) {
289 StringBuilder dirName = new StringBuilder(HConstants.HREGION_OLDLOGDIR_NAME);
290 if (conf.getBoolean(SEPARATE_OLDLOGDIR, DEFAULT_SEPARATE_OLDLOGDIR)) {
291 dirName.append(Path.SEPARATOR);
292 dirName.append(serverName);
294 return dirName.toString();
298 * Pulls a ServerName out of a Path generated according to our layout rules. In the below layouts,
299 * this method ignores the format of the logfile component. Current format: [base directory for
300 * hbase]/hbase/.logs/ServerName/logfile or [base directory for
301 * hbase]/hbase/.logs/ServerName-splitting/logfile Expected to work for individual log files and
302 * server-specific directories.
303 * @return null if it's not a log file. Returns the ServerName of the region server that created
304 * this log file otherwise.
306 public static ServerName getServerNameFromWALDirectoryName(Configuration conf, String path)
307 throws IOException {
308 if (path == null || path.length() <= HConstants.HREGION_LOGDIR_NAME.length()) {
309 return null;
312 if (conf == null) {
313 throw new IllegalArgumentException("parameter conf must be set");
316 final String rootDir = conf.get(HConstants.HBASE_DIR);
317 if (rootDir == null || rootDir.isEmpty()) {
318 throw new IllegalArgumentException(HConstants.HBASE_DIR + " key not found in conf.");
321 final StringBuilder startPathSB = new StringBuilder(rootDir);
322 if (!rootDir.endsWith("/")) {
323 startPathSB.append('/');
325 startPathSB.append(HConstants.HREGION_LOGDIR_NAME);
326 if (!HConstants.HREGION_LOGDIR_NAME.endsWith("/")) {
327 startPathSB.append('/');
329 final String startPath = startPathSB.toString();
331 String fullPath;
332 try {
333 fullPath = FileSystem.get(conf).makeQualified(new Path(path)).toString();
334 } catch (IllegalArgumentException e) {
335 LOG.info("Call to makeQualified failed on " + path + " " + e.getMessage());
336 return null;
339 if (!fullPath.startsWith(startPath)) {
340 return null;
343 final String serverNameAndFile = fullPath.substring(startPath.length());
345 if (serverNameAndFile.indexOf('/') < "a,0,0".length()) {
346 // Either it's a file (not a directory) or it's not a ServerName format
347 return null;
350 Path p = new Path(path);
351 return getServerNameFromWALDirectoryName(p);
355 * This function returns region server name from a log file name which is in one of the following
356 * formats:
357 * <ul>
358 * <li>hdfs://&lt;name node&gt;/hbase/.logs/&lt;server name&gt;-splitting/...</li>
359 * <li>hdfs://&lt;name node&gt;/hbase/.logs/&lt;server name&gt;/...</li>
360 * </ul>
361 * @return null if the passed in logFile isn't a valid WAL file path
363 public static ServerName getServerNameFromWALDirectoryName(Path logFile) {
364 String logDirName = logFile.getParent().getName();
365 // We were passed the directory and not a file in it.
366 if (logDirName.equals(HConstants.HREGION_LOGDIR_NAME)) {
367 logDirName = logFile.getName();
369 ServerName serverName = null;
370 if (logDirName.endsWith(SPLITTING_EXT)) {
371 logDirName = logDirName.substring(0, logDirName.length() - SPLITTING_EXT.length());
373 try {
374 serverName = ServerName.parseServerName(logDirName);
375 } catch (IllegalArgumentException|IllegalStateException ex) {
376 serverName = null;
377 LOG.warn("Cannot parse a server name from path=" + logFile + "; " + ex.getMessage());
379 if (serverName != null && serverName.getStartcode() < 0) {
380 LOG.warn("Invalid log file path=" + logFile);
381 serverName = null;
383 return serverName;
386 public static boolean isMetaFile(Path p) {
387 return isMetaFile(p.getName());
390 public static boolean isMetaFile(String p) {
391 if (p != null && p.endsWith(META_WAL_PROVIDER_ID)) {
392 return true;
394 return false;
397 public static boolean isArchivedLogFile(Path p) {
398 String oldLog = Path.SEPARATOR + HConstants.HREGION_OLDLOGDIR_NAME + Path.SEPARATOR;
399 return p.toString().contains(oldLog);
403 * Get the archived WAL file path
404 * @param path - active WAL file path
405 * @param conf - configuration
406 * @return archived path if exists, path - otherwise
407 * @throws IOException exception
409 public static Path getArchivedLogPath(Path path, Configuration conf) throws IOException {
410 Path rootDir = FSUtils.getRootDir(conf);
411 Path oldLogDir = new Path(rootDir, HConstants.HREGION_OLDLOGDIR_NAME);
412 if (conf.getBoolean(SEPARATE_OLDLOGDIR, DEFAULT_SEPARATE_OLDLOGDIR)) {
413 ServerName serverName = getServerNameFromWALDirectoryName(path);
414 if (serverName == null) {
415 LOG.error("Couldn't locate log: " + path);
416 return path;
418 oldLogDir = new Path(oldLogDir, serverName.getServerName());
420 Path archivedLogLocation = new Path(oldLogDir, path.getName());
421 final FileSystem fs = FSUtils.getCurrentFileSystem(conf);
423 if (fs.exists(archivedLogLocation)) {
424 LOG.info("Log " + path + " was moved to " + archivedLogLocation);
425 return archivedLogLocation;
426 } else {
427 LOG.error("Couldn't locate log: " + path);
428 return path;
433 * Opens WAL reader with retries and
434 * additional exception handling
435 * @param path path to WAL file
436 * @param conf configuration
437 * @return WAL Reader instance
438 * @throws IOException
440 public static org.apache.hadoop.hbase.wal.WAL.Reader
441 openReader(Path path, Configuration conf)
442 throws IOException
445 long retryInterval = 2000; // 2 sec
446 int maxAttempts = 30;
447 int attempt = 0;
448 Exception ee = null;
449 org.apache.hadoop.hbase.wal.WAL.Reader reader = null;
450 while (reader == null && attempt++ < maxAttempts) {
451 try {
452 // Detect if this is a new file, if so get a new reader else
453 // reset the current reader so that we see the new data
454 reader = WALFactory.createReader(path.getFileSystem(conf), path, conf);
455 return reader;
456 } catch (FileNotFoundException fnfe) {
457 // If the log was archived, continue reading from there
458 Path archivedLog = AbstractFSWALProvider.getArchivedLogPath(path, conf);
459 if (!Objects.equals(path, archivedLog)) {
460 return openReader(archivedLog, conf);
461 } else {
462 throw fnfe;
464 } catch (LeaseNotRecoveredException lnre) {
465 // HBASE-15019 the WAL was not closed due to some hiccup.
466 LOG.warn("Try to recover the WAL lease " + path, lnre);
467 recoverLease(conf, path);
468 reader = null;
469 ee = lnre;
470 } catch (NullPointerException npe) {
471 // Workaround for race condition in HDFS-4380
472 // which throws a NPE if we open a file before any data node has the most recent block
473 // Just sleep and retry. Will require re-reading compressed WALs for compressionContext.
474 LOG.warn("Got NPE opening reader, will retry.");
475 reader = null;
476 ee = npe;
478 if (reader == null) {
479 // sleep before next attempt
480 try {
481 Thread.sleep(retryInterval);
482 } catch (InterruptedException e) {
486 throw new IOException("Could not open reader", ee);
489 // For HBASE-15019
490 private static void recoverLease(final Configuration conf, final Path path) {
491 try {
492 final FileSystem dfs = FSUtils.getCurrentFileSystem(conf);
493 FSUtils fsUtils = FSUtils.getInstance(dfs, conf);
494 fsUtils.recoverFileLease(dfs, path, conf, new CancelableProgressable() {
495 @Override
496 public boolean progress() {
497 LOG.debug("Still trying to recover WAL lease: " + path);
498 return true;
501 } catch (IOException e) {
502 LOG.warn("unable to recover lease for WAL: " + path, e);
508 * Get prefix of the log from its name, assuming WAL name in format of
509 * log_prefix.filenumber.log_suffix
510 * @param name Name of the WAL to parse
511 * @return prefix of the log
512 * @see AbstractFSWAL#getCurrentFileName()
514 public static String getWALPrefixFromWALName(String name) {
515 int endIndex = name.replaceAll(META_WAL_PROVIDER_ID, "").lastIndexOf(".");
516 return name.substring(0, endIndex);