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.wal;
019
020import java.io.IOException;
021import java.util.ArrayList;
022import java.util.Collections;
023import java.util.Comparator;
024import java.util.List;
025import java.util.concurrent.atomic.AtomicBoolean;
026import java.util.concurrent.locks.ReadWriteLock;
027import java.util.concurrent.locks.ReentrantReadWriteLock;
028import java.util.regex.Pattern;
029import org.apache.hadoop.conf.Configuration;
030import org.apache.hadoop.fs.FileSystem;
031import org.apache.hadoop.fs.Path;
032import org.apache.hadoop.hbase.Abortable;
033import org.apache.hadoop.hbase.FailedCloseWALAfterInitializedErrorException;
034import org.apache.hadoop.hbase.HConstants;
035import org.apache.hadoop.hbase.ServerName;
036import org.apache.hadoop.hbase.client.RegionInfo;
037import org.apache.hadoop.hbase.regionserver.wal.AbstractFSWAL;
038import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
039import org.apache.hadoop.hbase.util.CancelableProgressable;
040import org.apache.hadoop.hbase.util.CommonFSUtils;
041import org.apache.hadoop.hbase.util.RecoverLeaseFSUtils;
042import org.apache.yetus.audience.InterfaceAudience;
043import org.apache.yetus.audience.InterfaceStability;
044import org.slf4j.Logger;
045import org.slf4j.LoggerFactory;
046
047import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
048
049/**
050 * Base class of a WAL Provider that returns a single thread safe WAL that writes to Hadoop FS. By
051 * default, this implementation picks a directory in Hadoop FS based on a combination of
052 * <ul>
053 * <li>the HBase root directory
054 * <li>HConstants.HREGION_LOGDIR_NAME
055 * <li>the given factory's factoryId (usually identifying the regionserver by host:port)
056 * </ul>
057 * It also uses the providerId to differentiate among files.
058 */
059@InterfaceAudience.Private
060@InterfaceStability.Evolving
061public abstract class AbstractFSWALProvider<T extends AbstractFSWAL<?>> implements WALProvider {
062
063  private static final Logger LOG = LoggerFactory.getLogger(AbstractFSWALProvider.class);
064
065  /** Separate old log into different dir by regionserver name **/
066  public static final String SEPARATE_OLDLOGDIR = "hbase.separate.oldlogdir.by.regionserver";
067  public static final boolean DEFAULT_SEPARATE_OLDLOGDIR = false;
068
069  public interface Initializer {
070    /**
071     * A method to initialize a WAL reader.
072     * @param startPosition the start position you want to read from, -1 means start reading from
073     *                      the first WAL entry. Notice that, the first entry is not started at
074     *                      position as we have several headers, so typically you should not pass 0
075     *                      here.
076     */
077    void init(FileSystem fs, Path path, Configuration c, long startPosition) throws IOException;
078  }
079
080  protected volatile T wal;
081  protected WALFactory factory;
082  protected Configuration conf;
083  protected List<WALActionsListener> listeners = new ArrayList<>();
084  protected String providerId;
085  protected AtomicBoolean initialized = new AtomicBoolean(false);
086  // for default wal provider, logPrefix won't change
087  protected String logPrefix;
088  protected Abortable abortable;
089
090  /**
091   * We use walCreateLock to prevent wal recreation in different threads, and also prevent getWALs
092   * missing the newly created WAL, see HBASE-21503 for more details.
093   */
094  private final ReadWriteLock walCreateLock = new ReentrantReadWriteLock();
095
096  /**
097   * @param factory    factory that made us, identity used for FS layout. may not be null
098   * @param conf       may not be null
099   * @param providerId differentiate between providers from one factory, used for FS layout. may be
100   *                   null
101   */
102  @Override
103  public void init(WALFactory factory, Configuration conf, String providerId, Abortable abortable)
104    throws IOException {
105    if (!initialized.compareAndSet(false, true)) {
106      throw new IllegalStateException("WALProvider.init should only be called once.");
107    }
108    this.factory = factory;
109    this.conf = conf;
110    this.providerId = providerId;
111    // get log prefix
112    StringBuilder sb = new StringBuilder().append(factory.factoryId);
113    if (providerId != null) {
114      if (providerId.startsWith(WAL_FILE_NAME_DELIMITER)) {
115        sb.append(providerId);
116      } else {
117        sb.append(WAL_FILE_NAME_DELIMITER).append(providerId);
118      }
119    }
120    logPrefix = sb.toString();
121    this.abortable = abortable;
122    doInit(conf);
123  }
124
125  @Override
126  public List<WAL> getWALs() {
127    if (wal != null) {
128      return Lists.newArrayList(wal);
129    }
130    walCreateLock.readLock().lock();
131    try {
132      if (wal == null) {
133        return Collections.emptyList();
134      } else {
135        return Lists.newArrayList(wal);
136      }
137    } finally {
138      walCreateLock.readLock().unlock();
139    }
140  }
141
142  @Override
143  public T getWAL(RegionInfo region) throws IOException {
144    T walCopy = wal;
145    if (walCopy != null) {
146      return walCopy;
147    }
148    walCreateLock.writeLock().lock();
149    try {
150      walCopy = wal;
151      if (walCopy != null) {
152        return walCopy;
153      }
154      walCopy = createWAL();
155      boolean succ = false;
156      try {
157        walCopy.init();
158        succ = true;
159      } finally {
160        if (!succ) {
161          try {
162            walCopy.close();
163          } catch (Throwable t) {
164            throw new FailedCloseWALAfterInitializedErrorException(
165              "Failed close after init wal failed.", t);
166          }
167        }
168      }
169      wal = walCopy;
170      return walCopy;
171    } finally {
172      walCreateLock.writeLock().unlock();
173    }
174  }
175
176  protected abstract T createWAL() throws IOException;
177
178  protected abstract void doInit(Configuration conf) throws IOException;
179
180  @Override
181  public void shutdown() throws IOException {
182    T log = this.wal;
183    if (log != null) {
184      log.shutdown();
185    }
186  }
187
188  @Override
189  public void close() throws IOException {
190    T log = this.wal;
191    if (log != null) {
192      log.close();
193    }
194  }
195
196  /**
197   * iff the given WALFactory is using the DefaultWALProvider for meta and/or non-meta, count the
198   * number of files (rolled and active). if either of them aren't, count 0 for that provider.
199   */
200  @Override
201  public long getNumLogFiles() {
202    T log = this.wal;
203    return log == null ? 0 : log.getNumLogFiles();
204  }
205
206  /**
207   * iff the given WALFactory is using the DefaultWALProvider for meta and/or non-meta, count the
208   * size of files (only rolled). if either of them aren't, count 0 for that provider.
209   */
210  @Override
211  public long getLogFileSize() {
212    T log = this.wal;
213    return log == null ? 0 : log.getLogFileSize();
214  }
215
216  /**
217   * returns the number of rolled WAL files.
218   */
219  public static int getNumRolledLogFiles(WAL wal) {
220    return ((AbstractFSWAL<?>) wal).getNumRolledLogFiles();
221  }
222
223  /**
224   * returns the size of rolled WAL files.
225   */
226  public static long getLogFileSize(WAL wal) {
227    return ((AbstractFSWAL<?>) wal).getLogFileSize();
228  }
229
230  /**
231   * return the current filename from the current wal.
232   */
233  public static Path getCurrentFileName(final WAL wal) {
234    return ((AbstractFSWAL<?>) wal).getCurrentFileName();
235  }
236
237  /**
238   * request a log roll, but don't actually do it.
239   */
240  static void requestLogRoll(final WAL wal) {
241    ((AbstractFSWAL<?>) wal).requestLogRoll();
242  }
243
244  // should be package private; more visible for use in AbstractFSWAL
245  public static final String WAL_FILE_NAME_DELIMITER = ".";
246  /** The hbase:meta region's WAL filename extension */
247  public static final String META_WAL_PROVIDER_ID = ".meta";
248  static final String DEFAULT_PROVIDER_ID = "default";
249
250  // Implementation details that currently leak in tests or elsewhere follow
251  /** File Extension used while splitting an WAL into regions (HBASE-2312) */
252  public static final String SPLITTING_EXT = "-splitting";
253
254  /**
255   * It returns the file create timestamp from the file name. For name format see
256   * {@link #validateWALFilename(String)} public until remaining tests move to o.a.h.h.wal
257   * @param wal must not be null
258   * @return the file number that is part of the WAL file name
259   */
260  public static long extractFileNumFromWAL(final WAL wal) {
261    final Path walName = ((AbstractFSWAL<?>) wal).getCurrentFileName();
262    if (walName == null) {
263      throw new IllegalArgumentException("The WAL path couldn't be null");
264    }
265    final String[] walPathStrs = walName.toString().split("\\" + WAL_FILE_NAME_DELIMITER);
266    return Long.parseLong(walPathStrs[walPathStrs.length - (isMetaFile(walName) ? 2 : 1)]);
267  }
268
269  /**
270   * Pattern used to validate a WAL file name see {@link #validateWALFilename(String)} for
271   * description.
272   */
273  private static final Pattern pattern =
274    Pattern.compile(".*\\.\\d*(" + META_WAL_PROVIDER_ID + ")*");
275
276  /**
277   * A WAL file name is of the format: &lt;wal-name&gt;{@link #WAL_FILE_NAME_DELIMITER}
278   * &lt;file-creation-timestamp&gt;[.meta]. provider-name is usually made up of a server-name and a
279   * provider-id
280   * @param filename name of the file to validate
281   * @return <tt>true</tt> if the filename matches an WAL, <tt>false</tt> otherwise
282   */
283  public static boolean validateWALFilename(String filename) {
284    return pattern.matcher(filename).matches();
285  }
286
287  /**
288   * Construct the directory name for all WALs on a given server. Dir names currently look like this
289   * for WALs: <code>hbase//WALs/kalashnikov.att.net,61634,1486865297088</code>.
290   * @param serverName Server name formatted as described in {@link ServerName}
291   * @return the relative WAL directory name, e.g. <code>.logs/1.example.org,60030,12345</code> if
292   *         <code>serverName</code> passed is <code>1.example.org,60030,12345</code>
293   */
294  public static String getWALDirectoryName(final String serverName) {
295    StringBuilder dirName = new StringBuilder(HConstants.HREGION_LOGDIR_NAME);
296    dirName.append("/");
297    dirName.append(serverName);
298    return dirName.toString();
299  }
300
301  /**
302   * Construct the directory name for all old WALs on a given server. The default old WALs dir looks
303   * like: <code>hbase/oldWALs</code>. If you config hbase.separate.oldlogdir.by.regionserver to
304   * true, it looks like <code>hbase//oldWALs/kalashnikov.att.net,61634,1486865297088</code>.
305   * @param serverName Server name formatted as described in {@link ServerName}
306   * @return the relative WAL directory name
307   */
308  public static String getWALArchiveDirectoryName(Configuration conf, final String serverName) {
309    StringBuilder dirName = new StringBuilder(HConstants.HREGION_OLDLOGDIR_NAME);
310    if (conf.getBoolean(SEPARATE_OLDLOGDIR, DEFAULT_SEPARATE_OLDLOGDIR)) {
311      dirName.append(Path.SEPARATOR);
312      dirName.append(serverName);
313    }
314    return dirName.toString();
315  }
316
317  /**
318   * Pulls a ServerName out of a Path generated according to our layout rules. In the below layouts,
319   * this method ignores the format of the logfile component. Current format: [base directory for
320   * hbase]/hbase/.logs/ServerName/logfile or [base directory for
321   * hbase]/hbase/.logs/ServerName-splitting/logfile Expected to work for individual log files and
322   * server-specific directories.
323   * @return null if it's not a log file. Returns the ServerName of the region server that created
324   *         this log file otherwise.
325   */
326  public static ServerName getServerNameFromWALDirectoryName(Configuration conf, String path)
327    throws IOException {
328    if (path == null || path.length() <= HConstants.HREGION_LOGDIR_NAME.length()) {
329      return null;
330    }
331
332    if (conf == null) {
333      throw new IllegalArgumentException("parameter conf must be set");
334    }
335
336    final String rootDir = conf.get(HConstants.HBASE_DIR);
337    if (rootDir == null || rootDir.isEmpty()) {
338      throw new IllegalArgumentException(HConstants.HBASE_DIR + " key not found in conf.");
339    }
340
341    final StringBuilder startPathSB = new StringBuilder(rootDir);
342    if (!rootDir.endsWith("/")) {
343      startPathSB.append('/');
344    }
345    startPathSB.append(HConstants.HREGION_LOGDIR_NAME);
346    if (!HConstants.HREGION_LOGDIR_NAME.endsWith("/")) {
347      startPathSB.append('/');
348    }
349    final String startPath = startPathSB.toString();
350
351    String fullPath;
352    try {
353      fullPath = FileSystem.get(conf).makeQualified(new Path(path)).toString();
354    } catch (IllegalArgumentException e) {
355      LOG.info("Call to makeQualified failed on " + path + " " + e.getMessage());
356      return null;
357    }
358
359    if (!fullPath.startsWith(startPath)) {
360      return null;
361    }
362
363    final String serverNameAndFile = fullPath.substring(startPath.length());
364
365    if (serverNameAndFile.indexOf('/') < "a,0,0".length()) {
366      // Either it's a file (not a directory) or it's not a ServerName format
367      return null;
368    }
369
370    Path p = new Path(path);
371    return getServerNameFromWALDirectoryName(p);
372  }
373
374  /**
375   * This function returns region server name from a log file name which is in one of the following
376   * formats:
377   * <ul>
378   * <li>hdfs://&lt;name node&gt;/hbase/.logs/&lt;server name&gt;-splitting/...</li>
379   * <li>hdfs://&lt;name node&gt;/hbase/.logs/&lt;server name&gt;/...</li>
380   * </ul>
381   * @return null if the passed in logFile isn't a valid WAL file path
382   */
383  public static ServerName getServerNameFromWALDirectoryName(Path logFile) {
384    String logDirName = logFile.getParent().getName();
385    // We were passed the directory and not a file in it.
386    if (logDirName.equals(HConstants.HREGION_LOGDIR_NAME)) {
387      logDirName = logFile.getName();
388    }
389    ServerName serverName = null;
390    if (logDirName.endsWith(SPLITTING_EXT)) {
391      logDirName = logDirName.substring(0, logDirName.length() - SPLITTING_EXT.length());
392    }
393    try {
394      serverName = ServerName.parseServerName(logDirName);
395    } catch (IllegalArgumentException | IllegalStateException ex) {
396      serverName = null;
397      LOG.warn("Cannot parse a server name from path={}", logFile, ex);
398    }
399    if (serverName != null && serverName.getStartCode() < 0) {
400      LOG.warn("Invalid log file path={}, start code {} is less than 0", logFile,
401        serverName.getStartCode());
402      serverName = null;
403    }
404    return serverName;
405  }
406
407  public static boolean isMetaFile(Path p) {
408    return isMetaFile(p.getName());
409  }
410
411  /** Returns True if String ends in {@link #META_WAL_PROVIDER_ID} */
412  public static boolean isMetaFile(String p) {
413    return p != null && p.endsWith(META_WAL_PROVIDER_ID);
414  }
415
416  /**
417   * Comparator used to compare WAL files together based on their start time. Just compares start
418   * times and nothing else.
419   */
420  public static class WALStartTimeComparator implements Comparator<Path> {
421    @Override
422    public int compare(Path o1, Path o2) {
423      return Long.compare(getTS(o1), getTS(o2));
424    }
425
426    /**
427     * Split a path to get the start time For example: 10.20.20.171%3A60020.1277499063250 Could also
428     * be a meta WAL which adds a '.meta' suffix or a synchronous replication WAL which adds a
429     * '.syncrep' suffix. Check.
430     * @param p path to split
431     * @return start time
432     */
433    public static long getTS(Path p) {
434      return WAL.getTimestamp(p.getName());
435    }
436  }
437
438  public static boolean isArchivedLogFile(Path p) {
439    String oldLog = Path.SEPARATOR + HConstants.HREGION_OLDLOGDIR_NAME + Path.SEPARATOR;
440    return p.toString().contains(oldLog);
441  }
442
443  /**
444   * Find the archived WAL file path if it is not able to locate in WALs dir.
445   * @param path - active WAL file path
446   * @param conf - configuration
447   * @return archived path if exists, null - otherwise
448   * @throws IOException exception
449   */
450  public static Path findArchivedLog(Path path, Configuration conf) throws IOException {
451    // If the path contains oldWALs keyword then exit early.
452    if (path.toString().contains(HConstants.HREGION_OLDLOGDIR_NAME)) {
453      return null;
454    }
455    Path walRootDir = CommonFSUtils.getWALRootDir(conf);
456    FileSystem fs = path.getFileSystem(conf);
457    // Try finding the log in old dir
458    Path oldLogDir = new Path(walRootDir, HConstants.HREGION_OLDLOGDIR_NAME);
459    Path archivedLogLocation = new Path(oldLogDir, path.getName());
460    if (fs.exists(archivedLogLocation)) {
461      LOG.info("Log " + path + " was moved to " + archivedLogLocation);
462      return archivedLogLocation;
463    }
464
465    ServerName serverName = getServerNameFromWALDirectoryName(path);
466    if (serverName == null) {
467      LOG.warn("Can not extract server name from path {}, "
468        + "give up searching the separated old log dir", path);
469      return null;
470    }
471    // Try finding the log in separate old log dir
472    oldLogDir = new Path(walRootDir, new StringBuilder(HConstants.HREGION_OLDLOGDIR_NAME)
473      .append(Path.SEPARATOR).append(serverName.getServerName()).toString());
474    archivedLogLocation = new Path(oldLogDir, path.getName());
475    if (fs.exists(archivedLogLocation)) {
476      LOG.info("Log " + path + " was moved to " + archivedLogLocation);
477      return archivedLogLocation;
478    }
479    LOG.error("Couldn't locate log: " + path);
480    return null;
481  }
482
483  // For HBASE-15019
484  public static void recoverLease(Configuration conf, Path path) {
485    try {
486      final FileSystem dfs = CommonFSUtils.getCurrentFileSystem(conf);
487      RecoverLeaseFSUtils.recoverFileLease(dfs, path, conf, new CancelableProgressable() {
488        @Override
489        public boolean progress() {
490          LOG.debug("Still trying to recover WAL lease: " + path);
491          return true;
492        }
493      });
494    } catch (IOException e) {
495      LOG.warn("unable to recover lease for WAL: " + path, e);
496    }
497  }
498
499  @Override
500  public void addWALActionsListener(WALActionsListener listener) {
501    listeners.add(listener);
502  }
503
504  /**
505   * Get prefix of the log from its name, assuming WAL name in format of
506   * log_prefix.filenumber.log_suffix
507   * @param name Name of the WAL to parse
508   * @return prefix of the log
509   * @see AbstractFSWAL#getCurrentFileName()
510   */
511  public static String getWALPrefixFromWALName(String name) {
512    int endIndex = name.replaceAll(META_WAL_PROVIDER_ID, "").lastIndexOf(".");
513    return name.substring(0, endIndex);
514  }
515}