001/**
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019
020package org.apache.hadoop.hbase.regionserver.wal;
021
022import java.io.IOException;
023import java.util.NavigableMap;
024
025import org.apache.hadoop.conf.Configuration;
026import org.apache.hadoop.fs.FileSystem;
027import org.apache.hadoop.fs.Path;
028import org.apache.hadoop.hbase.client.RegionInfo;
029import org.apache.hadoop.hbase.regionserver.MultiVersionConcurrencyControl;
030import org.apache.hadoop.hbase.util.CommonFSUtils;
031import org.apache.hadoop.hbase.wal.WAL;
032import org.apache.hadoop.hbase.wal.WALEdit;
033import org.apache.hadoop.hbase.wal.WALKeyImpl;
034import org.apache.yetus.audience.InterfaceAudience;
035import org.slf4j.Logger;
036import org.slf4j.LoggerFactory;
037import org.apache.hbase.thirdparty.com.google.protobuf.TextFormat;
038import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos;
039import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor;
040import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor;
041import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.RegionEventDescriptor;
042
043/**
044 * Helper methods to ease Region Server integration with the Write Ahead Log (WAL).
045 * Note that methods in this class specifically should not require access to anything
046 * other than the API found in {@link WAL}. For internal use only.
047 */
048@InterfaceAudience.Private
049public class WALUtil {
050  private static final Logger LOG = LoggerFactory.getLogger(WALUtil.class);
051
052  private WALUtil() {
053    // Shut down construction of this class.
054  }
055
056  /**
057   * Write the marker that a compaction has succeeded and is about to be committed.
058   * This provides info to the HMaster to allow it to recover the compaction if this regionserver
059   * dies in the middle. It also prevents the compaction from finishing if this regionserver has
060   * already lost its lease on the log.
061   *
062   * <p>This write is for internal use only. Not for external client consumption.
063   * @param mvcc Used by WAL to get sequence Id for the waledit.
064   */
065  public static WALKeyImpl writeCompactionMarker(WAL wal,
066      NavigableMap<byte[], Integer> replicationScope, RegionInfo hri, final CompactionDescriptor c,
067      MultiVersionConcurrencyControl mvcc)
068  throws IOException {
069    WALKeyImpl walKey =
070        writeMarker(wal, replicationScope, hri, WALEdit.createCompaction(hri, c), mvcc);
071    if (LOG.isTraceEnabled()) {
072      LOG.trace("Appended compaction marker " + TextFormat.shortDebugString(c));
073    }
074    return walKey;
075  }
076
077  /**
078   * Write a flush marker indicating a start / abort or a complete of a region flush
079   *
080   * <p>This write is for internal use only. Not for external client consumption.
081   */
082  public static WALKeyImpl writeFlushMarker(WAL wal, NavigableMap<byte[], Integer> replicationScope,
083      RegionInfo hri, final FlushDescriptor f, boolean sync, MultiVersionConcurrencyControl mvcc)
084          throws IOException {
085    WALKeyImpl walKey = doFullAppendTransaction(wal, replicationScope, hri,
086        WALEdit.createFlushWALEdit(hri, f), mvcc, sync);
087    if (LOG.isTraceEnabled()) {
088      LOG.trace("Appended flush marker " + TextFormat.shortDebugString(f));
089    }
090    return walKey;
091  }
092
093  /**
094   * Write a region open marker indicating that the region is opened.
095   * This write is for internal use only. Not for external client consumption.
096   */
097  public static WALKeyImpl writeRegionEventMarker(WAL wal,
098      NavigableMap<byte[], Integer> replicationScope, RegionInfo hri,
099      final RegionEventDescriptor r, final MultiVersionConcurrencyControl mvcc)
100  throws IOException {
101    WALKeyImpl walKey = writeMarker(wal, replicationScope, hri,
102        WALEdit.createRegionEventWALEdit(hri, r), mvcc);
103    if (LOG.isTraceEnabled()) {
104      LOG.trace("Appended region event marker " + TextFormat.shortDebugString(r));
105    }
106    return walKey;
107  }
108
109  /**
110   * Write a log marker that a bulk load has succeeded and is about to be committed.
111   * This write is for internal use only. Not for external client consumption.
112   * @param wal The log to write into.
113   * @param replicationScope The replication scope of the families in the HRegion
114   * @param hri A description of the region in the table that we are bulk loading into.
115   * @param desc A protocol buffers based description of the client's bulk loading request
116   * @return walKey with sequenceid filled out for this bulk load marker
117   * @throws IOException We will throw an IOException if we can not append to the HLog.
118   */
119  public static WALKeyImpl writeBulkLoadMarkerAndSync(final WAL wal,
120      final NavigableMap<byte[], Integer> replicationScope, final RegionInfo hri,
121      final WALProtos.BulkLoadDescriptor desc, final MultiVersionConcurrencyControl mvcc)
122          throws IOException {
123    WALKeyImpl walKey =
124        writeMarker(wal, replicationScope, hri, WALEdit.createBulkLoadEvent(hri, desc), mvcc);
125    if (LOG.isTraceEnabled()) {
126      LOG.trace("Appended Bulk Load marker " + TextFormat.shortDebugString(desc));
127    }
128    return walKey;
129  }
130
131  private static WALKeyImpl writeMarker(final WAL wal,
132      final NavigableMap<byte[], Integer> replicationScope, final RegionInfo hri,
133      final WALEdit edit, final MultiVersionConcurrencyControl mvcc)
134  throws IOException {
135    // If sync == true in below, then timeout is not used; safe to pass UNSPECIFIED_TIMEOUT
136    return doFullAppendTransaction(wal, replicationScope, hri, edit, mvcc, true);
137  }
138
139  /**
140   * A 'full' WAL transaction involves starting an mvcc transaction followed by an append,
141   * an optional sync, and then a call to complete the mvcc transaction. This method does it all.
142   * Good for case of adding a single edit or marker to the WAL.
143   *
144   * <p>This write is for internal use only. Not for external client consumption.
145   * @return WALKeyImpl that was added to the WAL.
146   */
147  public static WALKeyImpl doFullAppendTransaction(final WAL wal,
148      final NavigableMap<byte[], Integer> replicationScope, final RegionInfo hri,
149      final WALEdit edit, final MultiVersionConcurrencyControl mvcc, final boolean sync)
150  throws IOException {
151    // TODO: Pass in current time to use?
152    WALKeyImpl walKey = new WALKeyImpl(hri.getEncodedNameAsBytes(), hri.getTable(),
153        System.currentTimeMillis(), mvcc, replicationScope);
154    long trx = MultiVersionConcurrencyControl.NONE;
155    try {
156      trx = wal.append(hri, walKey, edit, false);
157      if (sync) {
158        wal.sync(trx);
159      }
160      // Call complete only here because these are markers only. They are not for clients to read.
161      mvcc.complete(walKey.getWriteEntry());
162    } catch (IOException ioe) {
163      if (walKey.getWriteEntry() != null) {
164        mvcc.complete(walKey.getWriteEntry());
165      }
166      throw ioe;
167    }
168    return walKey;
169  }
170
171  /**
172   * Blocksize returned here is 2x the default HDFS blocksize unless explicitly set in
173   * Configuration. Works in tandem with hbase.regionserver.logroll.multiplier. See comment in
174   * AbstractFSWAL in Constructor where we set blocksize and logrollsize for why.
175   * @return Blocksize to use writing WALs.
176   */
177  public static long getWALBlockSize(Configuration conf, FileSystem fs, Path dir)
178      throws IOException {
179    return conf.getLong("hbase.regionserver.hlog.blocksize",
180        CommonFSUtils.getDefaultBlockSize(fs, dir) * 2);
181  }
182}