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 */
019package org.apache.hadoop.hbase.wal;
020
021import java.io.IOException;
022import java.util.ArrayList;
023import java.util.List;
024import java.util.Map;
025import java.util.Set;
026import java.util.TreeSet;
027
028import org.apache.hadoop.hbase.Cell;
029import org.apache.hadoop.hbase.CellUtil;
030import org.apache.hadoop.hbase.HBaseInterfaceAudience;
031import org.apache.hadoop.hbase.PrivateCellUtil;
032import org.apache.hadoop.hbase.KeyValue;
033import org.apache.hadoop.hbase.client.RegionInfo;
034import org.apache.hadoop.hbase.codec.Codec;
035import org.apache.hadoop.hbase.io.HeapSize;
036import org.apache.hadoop.hbase.util.Bytes;
037import org.apache.hadoop.hbase.util.ClassSize;
038import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
039import org.apache.yetus.audience.InterfaceAudience;
040import org.slf4j.Logger;
041import org.slf4j.LoggerFactory;
042import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
043import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos;
044import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor;
045import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor;
046import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.RegionEventDescriptor;
047
048
049/**
050 * Used in HBase's transaction log (WAL) to represent a collection of edits (Cell/KeyValue objects)
051 * that came in as a single transaction. All the edits for a given transaction are written out as a
052 * single record, in PB format, followed (optionally) by Cells written via the WALCellEncoder.
053 * <p>This class is LimitedPrivate for CPs to read-only. The {@link #add} methods are
054 * classified as private methods, not for use by CPs.</p>
055 * <p>WALEdit will accumulate a Set of all column family names referenced by the Cells
056 * {@link #add(Cell)}'d. This is an optimization. Usually when loading a WALEdit, we have the
057 * column family name to-hand.. just shove it into the WALEdit if available. Doing this, we can
058 * save on a parse of each Cell to figure column family down the line when we go to add the
059 * WALEdit to the WAL file. See the hand-off in FSWALEntry Constructor.
060 */
061// TODO: Do not expose this class to Coprocessors. It has set methods. A CP might meddle.
062@InterfaceAudience.LimitedPrivate({ HBaseInterfaceAudience.REPLICATION,
063    HBaseInterfaceAudience.COPROC })
064public class WALEdit implements HeapSize {
065  private static final Logger LOG = LoggerFactory.getLogger(WALEdit.class);
066
067  // TODO: Get rid of this; see HBASE-8457
068  public static final byte [] METAFAMILY = Bytes.toBytes("METAFAMILY");
069  @VisibleForTesting
070  public static final byte [] METAROW = Bytes.toBytes("METAROW");
071  @VisibleForTesting
072  public static final byte[] COMPACTION = Bytes.toBytes("HBASE::COMPACTION");
073  @VisibleForTesting
074  public static final byte [] FLUSH = Bytes.toBytes("HBASE::FLUSH");
075  @VisibleForTesting
076  public static final byte [] REGION_EVENT = Bytes.toBytes("HBASE::REGION_EVENT");
077  @VisibleForTesting
078  public static final byte [] BULK_LOAD = Bytes.toBytes("HBASE::BULK_LOAD");
079
080  private final boolean replay;
081
082  private ArrayList<Cell> cells = null;
083
084  /**
085   * All the Cell families in <code>cells</code>. Updated by {@link #add(Cell)} and
086   * {@link #add(Map)}. This Set is passed to the FSWALEntry so it does not have
087   * to recalculate the Set of families in a transaction; makes for a bunch of CPU savings.
088   * An optimization that saves on CPU-expensive Cell-parsing.
089   */
090  private Set<byte []> families = null;
091
092  public WALEdit() {
093    this(false);
094  }
095
096  /**
097   * @deprecated since 2.0.1 and will be removed in 4.0.0. Use {@link #WALEdit(int, boolean)}
098   *   instead.
099   * @see #WALEdit(int, boolean)
100   * @see <a href="https://issues.apache.org/jira/browse/HBASE-20781">HBASE-20781</a>
101   */
102  @Deprecated
103  public WALEdit(boolean isReplay) {
104    this(1, isReplay);
105  }
106
107  /**
108   * @deprecated since 2.0.1 and will be removed in 4.0.0. Use {@link #WALEdit(int, boolean)}
109   *   instead.
110   * @see #WALEdit(int, boolean)
111   * @see <a href="https://issues.apache.org/jira/browse/HBASE-20781">HBASE-20781</a>
112   */
113  @Deprecated
114  public WALEdit(int cellCount) {
115    this(cellCount, false);
116  }
117
118  /**
119   * @param cellCount Pass so can pre-size the WALEdit. Optimization.
120   */
121  public WALEdit(int cellCount, boolean isReplay) {
122    this.replay = isReplay;
123    cells = new ArrayList<>(cellCount);
124  }
125
126  private Set<byte[]> getOrCreateFamilies() {
127    if (this.families == null) {
128      this.families = new TreeSet<byte []>(Bytes.BYTES_COMPARATOR);
129    }
130    return this.families;
131  }
132
133  /**
134   * For use by FSWALEntry ONLY. An optimization.
135   * @return All families in {@link #getCells()}; may be null.
136   */
137  public Set<byte []> getFamilies() {
138    return this.families;
139  }
140
141  /**
142   * @return True is <code>f</code> is {@link #METAFAMILY}
143   */
144  public static boolean isMetaEditFamily(final byte [] f) {
145    return Bytes.equals(METAFAMILY, f);
146  }
147
148  public static boolean isMetaEditFamily(Cell cell) {
149    return CellUtil.matchingFamily(cell, METAFAMILY);
150  }
151
152  public boolean isMetaEdit() {
153    for (Cell cell: cells) {
154      if (!isMetaEditFamily(cell)) {
155        return false;
156      }
157    }
158    return true;
159  }
160
161  /**
162   * @return True when current WALEdit is created by log replay. Replication skips WALEdits from
163   *         replay.
164   */
165  public boolean isReplay() {
166    return this.replay;
167  }
168
169  @InterfaceAudience.Private
170  public WALEdit add(Cell cell, byte [] family) {
171    getOrCreateFamilies().add(family);
172    return addCell(cell);
173  }
174
175  @InterfaceAudience.Private
176  public WALEdit add(Cell cell) {
177    // We clone Family each time we add a Cell. Expensive but safe. For CPU savings, use
178    // add(Map) or add(Cell, family).
179    return add(cell, CellUtil.cloneFamily(cell));
180  }
181
182  public boolean isEmpty() {
183    return cells.isEmpty();
184  }
185
186  public int size() {
187    return cells.size();
188  }
189
190  public ArrayList<Cell> getCells() {
191    return cells;
192  }
193
194  /**
195   * This is not thread safe.
196   * This will change the WALEdit and shouldn't be used unless you are sure that nothing
197   * else depends on the contents being immutable.
198   *
199   * @param cells the list of cells that this WALEdit now contains.
200   */
201  @InterfaceAudience.Private
202  // Used by replay.
203  public void setCells(ArrayList<Cell> cells) {
204    this.cells = cells;
205    this.families = null;
206  }
207
208  /**
209   * Reads WALEdit from cells.
210   * @param cellDecoder Cell decoder.
211   * @param expectedCount Expected cell count.
212   * @return Number of KVs read.
213   */
214  public int readFromCells(Codec.Decoder cellDecoder, int expectedCount) throws IOException {
215    cells.clear();
216    cells.ensureCapacity(expectedCount);
217    while (cells.size() < expectedCount && cellDecoder.advance()) {
218      cells.add(cellDecoder.current());
219    }
220    return cells.size();
221  }
222
223  @Override
224  public long heapSize() {
225    long ret = ClassSize.ARRAYLIST;
226    for (Cell cell : cells) {
227      ret += PrivateCellUtil.estimatedSizeOfCell(cell);
228    }
229    return ret;
230  }
231
232  public long estimatedSerializedSizeOf() {
233    long ret = 0;
234    for (Cell cell: cells) {
235      ret += PrivateCellUtil.estimatedSerializedSizeOf(cell);
236    }
237    return ret;
238  }
239
240  @Override
241  public String toString() {
242    StringBuilder sb = new StringBuilder();
243
244    sb.append("[#edits: " + cells.size() + " = <");
245    for (Cell cell : cells) {
246      sb.append(cell);
247      sb.append("; ");
248    }
249    sb.append(">]");
250    return sb.toString();
251  }
252
253  public static WALEdit createFlushWALEdit(RegionInfo hri, FlushDescriptor f) {
254    KeyValue kv = new KeyValue(getRowForRegion(hri), METAFAMILY, FLUSH,
255      EnvironmentEdgeManager.currentTime(), f.toByteArray());
256    return new WALEdit().add(kv, METAFAMILY);
257  }
258
259  public static FlushDescriptor getFlushDescriptor(Cell cell) throws IOException {
260    if (CellUtil.matchingColumn(cell, METAFAMILY, FLUSH)) {
261      return FlushDescriptor.parseFrom(CellUtil.cloneValue(cell));
262    }
263    return null;
264  }
265
266  public static WALEdit createRegionEventWALEdit(RegionInfo hri,
267      RegionEventDescriptor regionEventDesc) {
268    KeyValue kv = new KeyValue(getRowForRegion(hri), METAFAMILY, REGION_EVENT,
269      EnvironmentEdgeManager.currentTime(), regionEventDesc.toByteArray());
270    return new WALEdit().add(kv, METAFAMILY);
271  }
272
273  public static RegionEventDescriptor getRegionEventDescriptor(Cell cell) throws IOException {
274    if (CellUtil.matchingColumn(cell, METAFAMILY, REGION_EVENT)) {
275      return RegionEventDescriptor.parseFrom(CellUtil.cloneValue(cell));
276    }
277    return null;
278  }
279
280  /**
281   * Create a compaction WALEdit
282   * @param c
283   * @return A WALEdit that has <code>c</code> serialized as its value
284   */
285  public static WALEdit createCompaction(final RegionInfo hri, final CompactionDescriptor c) {
286    byte [] pbbytes = c.toByteArray();
287    KeyValue kv = new KeyValue(getRowForRegion(hri), METAFAMILY, COMPACTION,
288      EnvironmentEdgeManager.currentTime(), pbbytes);
289    return new WALEdit().add(kv, METAFAMILY); //replication scope null so this won't be replicated
290  }
291
292  public static byte[] getRowForRegion(RegionInfo hri) {
293    byte[] startKey = hri.getStartKey();
294    if (startKey.length == 0) {
295      // empty row key is not allowed in mutations because it is both the start key and the end key
296      // we return the smallest byte[] that is bigger (in lex comparison) than byte[0].
297      return new byte[] {0};
298    }
299    return startKey;
300  }
301
302  /**
303   * Deserialized and returns a CompactionDescriptor is the KeyValue contains one.
304   * @param kv the key value
305   * @return deserialized CompactionDescriptor or null.
306   */
307  public static CompactionDescriptor getCompaction(Cell kv) throws IOException {
308    if (isCompactionMarker(kv)) {
309      return CompactionDescriptor.parseFrom(CellUtil.cloneValue(kv));
310    }
311    return null;
312  }
313
314  /**
315   * Returns true if the given cell is a serialized {@link CompactionDescriptor}
316   *
317   * @see #getCompaction(Cell)
318   */
319  public static boolean isCompactionMarker(Cell cell) {
320    return CellUtil.matchingColumn(cell, METAFAMILY, COMPACTION);
321  }
322
323  /**
324   * Create a bulk loader WALEdit
325   *
326   * @param hri                The RegionInfo for the region in which we are bulk loading
327   * @param bulkLoadDescriptor The descriptor for the Bulk Loader
328   * @return The WALEdit for the BulkLoad
329   */
330  public static WALEdit createBulkLoadEvent(RegionInfo hri,
331                                            WALProtos.BulkLoadDescriptor bulkLoadDescriptor) {
332    KeyValue kv = new KeyValue(getRowForRegion(hri),
333        METAFAMILY,
334        BULK_LOAD,
335        EnvironmentEdgeManager.currentTime(),
336        bulkLoadDescriptor.toByteArray());
337    return new WALEdit().add(kv, METAFAMILY);
338  }
339
340  /**
341   * Deserialized and returns a BulkLoadDescriptor from the passed in Cell
342   * @param cell the key value
343   * @return deserialized BulkLoadDescriptor or null.
344   */
345  public static WALProtos.BulkLoadDescriptor getBulkLoadDescriptor(Cell cell) throws IOException {
346    if (CellUtil.matchingColumn(cell, METAFAMILY, BULK_LOAD)) {
347      return WALProtos.BulkLoadDescriptor.parseFrom(CellUtil.cloneValue(cell));
348    }
349    return null;
350  }
351
352  /**
353   * Append the given map of family->edits to a WALEdit data structure.
354   * This does not write to the WAL itself.
355   * Note that as an optimization, we will stamp the Set of column families into the WALEdit
356   * to save on our having to calculate it subsequently way down in the actual WAL writing.
357   *
358   * @param familyMap map of family->edits
359   */
360  public void add(Map<byte[], List<Cell>> familyMap) {
361    for (Map.Entry<byte [], List<Cell>> e: familyMap.entrySet()) {
362      // 'foreach' loop NOT used. See HBASE-12023 "...creates too many iterator objects."
363      int listSize = e.getValue().size();
364      // Add all Cells first and then at end, add the family rather than call {@link #add(Cell)}
365      // and have it clone family each time. Optimization!
366      for (int i = 0; i < listSize; i++) {
367        addCell(e.getValue().get(i));
368      }
369      addFamily(e.getKey());
370    }
371  }
372
373  private void addFamily(byte [] family) {
374    getOrCreateFamilies().add(family);
375  }
376
377  private WALEdit addCell(Cell cell) {
378    this.cells.add(cell);
379    return this;
380  }
381}