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.regionserver.wal;
019
020import java.io.IOException;
021import java.util.Collections;
022import java.util.List;
023import java.util.Set;
024import java.util.TreeSet;
025import org.apache.hadoop.hbase.Cell;
026import org.apache.hadoop.hbase.CellUtil;
027import org.apache.hadoop.hbase.PrivateCellUtil;
028import org.apache.hadoop.hbase.client.RegionInfo;
029import org.apache.hadoop.hbase.ipc.ServerCall;
030import org.apache.hadoop.hbase.regionserver.MultiVersionConcurrencyControl;
031import org.apache.hadoop.hbase.util.Bytes;
032import org.apache.hadoop.hbase.wal.WAL.Entry;
033import org.apache.hadoop.hbase.wal.WALEdit;
034import org.apache.hadoop.hbase.wal.WALKeyImpl;
035import org.apache.yetus.audience.InterfaceAudience;
036
037import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
038import org.apache.hbase.thirdparty.org.apache.commons.collections4.CollectionUtils;
039
040/**
041 * A WAL Entry for {@link AbstractFSWAL} implementation.  Immutable.
042 * A subclass of {@link Entry} that carries extra info across the ring buffer such as
043 * region sequenceid (we want to use this later, just before we write the WAL to ensure region
044 * edits maintain order).  The extra info added here is not 'serialized' as part of the WALEdit
045 * hence marked 'transient' to underline this fact.  It also adds mechanism so we can wait on
046 * the assign of the region sequence id.  See #stampRegionSequenceId().
047 */
048@InterfaceAudience.Private
049class FSWALEntry extends Entry {
050  // The below data members are denoted 'transient' just to highlight these are not persisted;
051  // they are only in memory and held here while passing over the ring buffer.
052  private final transient long txid;
053
054  /**
055   * If false, means this is a meta edit written by the hbase system itself. It was not in
056   * memstore. HBase uses these edit types to note in the log operational transitions such
057   * as compactions, flushes, or region open/closes.
058   */
059  private final transient boolean inMemstore;
060
061  /**
062   * Set if this is a meta edit and it is of close region type.
063   */
064  private final transient boolean closeRegion;
065
066  private final transient RegionInfo regionInfo;
067  private final transient Set<byte[]> familyNames;
068  private final transient ServerCall<?> rpcCall;
069
070  /**
071   * @param inMemstore If true, then this is a data edit, one that came from client. If false, it
072   *   is a meta edit made by the hbase system itself and is for the WAL only.
073   */
074  FSWALEntry(final long txid, final WALKeyImpl key, final WALEdit edit, final RegionInfo regionInfo,
075    final boolean inMemstore, ServerCall<?> rpcCall) {
076    super(key, edit);
077    this.inMemstore = inMemstore;
078    this.closeRegion = !inMemstore && edit.isRegionCloseMarker();
079    this.regionInfo = regionInfo;
080    this.txid = txid;
081    if (inMemstore) {
082      // construct familyNames here to reduce the work of log sinker.
083      Set<byte[]> families = edit.getFamilies();
084      this.familyNames = families != null ? families : collectFamilies(edit.getCells());
085    } else {
086      this.familyNames = Collections.emptySet();
087    }
088    this.rpcCall = rpcCall;
089    if (rpcCall != null) {
090      rpcCall.retainByWAL();
091    }
092  }
093
094  @VisibleForTesting
095  static Set<byte[]> collectFamilies(List<Cell> cells) {
096    if (CollectionUtils.isEmpty(cells)) {
097      return Collections.emptySet();
098    } else {
099      Set<byte[]> set = new TreeSet<>(Bytes.BYTES_COMPARATOR);
100      for (Cell cell: cells) {
101        if (!WALEdit.isMetaEditFamily(cell)) {
102          set.add(CellUtil.cloneFamily(cell));
103        }
104      }
105      return set;
106    }
107  }
108
109  @Override
110  public String toString() {
111    return "sequence=" + this.txid + ", " + super.toString();
112  }
113
114  boolean isInMemStore() {
115    return this.inMemstore;
116  }
117
118  boolean isCloseRegion() {
119    return closeRegion;
120  }
121
122  RegionInfo getRegionInfo() {
123    return this.regionInfo;
124  }
125
126  /**
127   * @return The transaction id of this edit.
128   */
129  long getTxid() {
130    return this.txid;
131  }
132
133  /**
134   * Here is where a WAL edit gets its sequenceid. SIDE-EFFECT is our stamping the sequenceid into
135   * every Cell AND setting the sequenceid into the MVCC WriteEntry!!!!
136   * @return The sequenceid we stamped on this edit.
137   */
138  long stampRegionSequenceId(MultiVersionConcurrencyControl.WriteEntry we) throws IOException {
139    long regionSequenceId = we.getWriteNumber();
140    if (!this.getEdit().isReplay() && inMemstore) {
141      for (Cell c : getEdit().getCells()) {
142        PrivateCellUtil.setSequenceId(c, regionSequenceId);
143      }
144    }
145
146    getKey().setWriteEntry(we);
147    return regionSequenceId;
148  }
149
150  /**
151   * @return the family names which are effected by this edit.
152   */
153  Set<byte[]> getFamilyNames() {
154    return familyNames;
155  }
156
157  void release() {
158    if (rpcCall != null) {
159      rpcCall.releaseByWAL();
160    }
161  }
162}