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.HashMap;
023import java.util.List;
024import java.util.Map;
025import java.util.NavigableMap;
026import java.util.TreeMap;
027import java.util.UUID;
028
029import org.apache.hadoop.hbase.HBaseInterfaceAudience;
030import org.apache.hadoop.hbase.HConstants;
031import org.apache.hadoop.hbase.TableName;
032import org.apache.hadoop.hbase.regionserver.MultiVersionConcurrencyControl;
033import org.apache.hadoop.hbase.regionserver.SequenceId;
034import org.apache.hadoop.hbase.regionserver.wal.CompressionContext;
035import org.apache.hadoop.hbase.regionserver.wal.WALCellCodec;
036import org.apache.hadoop.hbase.util.Bytes;
037import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
038import org.apache.yetus.audience.InterfaceAudience;
039
040import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
041import org.apache.hbase.thirdparty.com.google.protobuf.ByteString;
042
043import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
044import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos;
045import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FamilyScope;
046import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.ScopeType;
047
048/**
049 * Default implementation of Key for an Entry in the WAL.
050 * For internal use only though Replication needs to have access.
051 *
052 * The log intermingles edits to many tables and rows, so each log entry
053 * identifies the appropriate table and row.  Within a table and row, they're
054 * also sorted.
055 *
056 * <p>Some Transactional edits (START, COMMIT, ABORT) will not have an associated row.
057 *
058 */
059// TODO: Key and WALEdit are never used separately, or in one-to-many relation, for practical
060//       purposes. They need to be merged into WALEntry.
061@InterfaceAudience.LimitedPrivate({HBaseInterfaceAudience.REPLICATION})
062public class WALKeyImpl implements WALKey {
063  public static final WALKeyImpl EMPTY_WALKEYIMPL = new WALKeyImpl();
064
065  public MultiVersionConcurrencyControl getMvcc() {
066    return mvcc;
067  }
068
069  /**
070   * Use it to complete mvcc transaction. This WALKeyImpl was part of
071   * (the transaction is started when you call append; see the comment on FSHLog#append). To
072   * complete call
073   * {@link MultiVersionConcurrencyControl#complete(MultiVersionConcurrencyControl.WriteEntry)}
074   * or {@link MultiVersionConcurrencyControl#complete(MultiVersionConcurrencyControl.WriteEntry)}
075   * @return A WriteEntry gotten from local WAL subsystem.
076   * @see #setWriteEntry(MultiVersionConcurrencyControl.WriteEntry)
077   */
078  public MultiVersionConcurrencyControl.WriteEntry getWriteEntry() {
079    return this.writeEntry;
080  }
081
082  public void setWriteEntry(MultiVersionConcurrencyControl.WriteEntry writeEntry) {
083    assert this.writeEntry == null;
084    this.writeEntry = writeEntry;
085    // Set our sequenceid now using WriteEntry.
086    this.sequenceId = writeEntry.getWriteNumber();
087  }
088
089  private byte [] encodedRegionName;
090
091  private TableName tablename;
092
093  /**
094   * SequenceId for this edit. Set post-construction at write-to-WAL time. Until then it is
095   * NO_SEQUENCE_ID. Change it so multiple threads can read it -- e.g. access is synchronized.
096   */
097  private long sequenceId;
098
099  /**
100   * Used during WAL replay; the sequenceId of the edit when it came into the system.
101   */
102  private long origLogSeqNum = 0;
103
104  /** Time at which this edit was written. */
105  private long writeTime;
106
107  /** The first element in the list is the cluster id on which the change has originated */
108  private List<UUID> clusterIds;
109
110  private NavigableMap<byte[], Integer> replicationScope;
111
112  private long nonceGroup = HConstants.NO_NONCE;
113  private long nonce = HConstants.NO_NONCE;
114  private MultiVersionConcurrencyControl mvcc;
115
116  /**
117   * Set in a way visible to multiple threads; e.g. synchronized getter/setters.
118   */
119  private MultiVersionConcurrencyControl.WriteEntry writeEntry;
120
121  private Map<String, byte[]> extendedAttributes;
122
123  public WALKeyImpl() {
124    init(null, null, 0L, HConstants.LATEST_TIMESTAMP,
125        new ArrayList<>(), HConstants.NO_NONCE, HConstants.NO_NONCE, null, null, null);
126  }
127
128  public WALKeyImpl(final NavigableMap<byte[], Integer> replicationScope) {
129    init(null, null, 0L, HConstants.LATEST_TIMESTAMP,
130        new ArrayList<>(), HConstants.NO_NONCE, HConstants.NO_NONCE, null, replicationScope, null);
131  }
132
133  @VisibleForTesting
134  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename,
135                long logSeqNum,
136      final long now, UUID clusterId) {
137    List<UUID> clusterIds = new ArrayList<>(1);
138    clusterIds.add(clusterId);
139    init(encodedRegionName, tablename, logSeqNum, now, clusterIds, HConstants.NO_NONCE,
140      HConstants.NO_NONCE, null, null, null);
141  }
142
143  // TODO: Fix being able to pass in sequenceid.
144  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename, final long now) {
145    init(encodedRegionName,
146        tablename,
147        NO_SEQUENCE_ID,
148        now,
149        EMPTY_UUIDS,
150        HConstants.NO_NONCE,
151        HConstants.NO_NONCE,
152        null, null, null);
153  }
154
155  // TODO: Fix being able to pass in sequenceid.
156  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename, final long now,
157      final NavigableMap<byte[], Integer> replicationScope) {
158    init(encodedRegionName, tablename, NO_SEQUENCE_ID, now, EMPTY_UUIDS, HConstants.NO_NONCE,
159        HConstants.NO_NONCE, null, replicationScope, null);
160  }
161
162  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename, final long now,
163      MultiVersionConcurrencyControl mvcc, final NavigableMap<byte[], Integer> replicationScope) {
164    init(encodedRegionName, tablename, NO_SEQUENCE_ID, now, EMPTY_UUIDS, HConstants.NO_NONCE,
165        HConstants.NO_NONCE, mvcc, replicationScope, null);
166  }
167
168  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename, final long now,
169                    MultiVersionConcurrencyControl mvcc,
170                    final NavigableMap<byte[], Integer> replicationScope,
171                    Map<String, byte[]> extendedAttributes) {
172    init(encodedRegionName, tablename, NO_SEQUENCE_ID, now, EMPTY_UUIDS, HConstants.NO_NONCE,
173        HConstants.NO_NONCE, mvcc, replicationScope, extendedAttributes);
174  }
175
176  public WALKeyImpl(final byte[] encodedRegionName,
177                final TableName tablename,
178                final long now,
179                MultiVersionConcurrencyControl mvcc) {
180    init(encodedRegionName,
181        tablename,
182        NO_SEQUENCE_ID,
183        now,
184        EMPTY_UUIDS,
185        HConstants.NO_NONCE,
186        HConstants.NO_NONCE,
187        mvcc, null, null);
188  }
189
190  /**
191   * Copy constructor that takes in an existing WALKeyImpl plus some extended attributes.
192   * Intended for coprocessors to add annotations to a system-generated WALKey
193   * for persistence to the WAL.
194   * @param key Key to be copied into this new key
195   * @param extendedAttributes Extra attributes to copy into the new key
196   */
197  public WALKeyImpl(WALKeyImpl key,
198                    Map<String, byte[]> extendedAttributes){
199    init(key.getEncodedRegionName(), key.getTableName(), key.getSequenceId(),
200        key.getWriteTime(), key.getClusterIds(), key.getNonceGroup(), key.getNonce(),
201        key.getMvcc(), key.getReplicationScopes(), extendedAttributes);
202
203  }
204
205  /**
206   * Copy constructor that takes in an existing WALKey, the extra WALKeyImpl fields that the
207   * parent interface is missing, plus some extended attributes. Intended
208   * for coprocessors to add annotations to a system-generated WALKey for
209   * persistence to the WAL.
210   */
211  public WALKeyImpl(WALKey key,
212                    List<UUID> clusterIds,
213                    MultiVersionConcurrencyControl mvcc,
214                    final NavigableMap<byte[], Integer> replicationScopes,
215                    Map<String, byte[]> extendedAttributes){
216    init(key.getEncodedRegionName(), key.getTableName(), key.getSequenceId(),
217        key.getWriteTime(), clusterIds, key.getNonceGroup(), key.getNonce(),
218        mvcc, replicationScopes, extendedAttributes);
219
220  }
221  /**
222   * Create the log key for writing to somewhere.
223   * We maintain the tablename mainly for debugging purposes.
224   * A regionName is always a sub-table object.
225   * <p>Used by log splitting and snapshots.
226   *
227   * @param encodedRegionName Encoded name of the region as returned by
228   *                         <code>HRegionInfo#getEncodedNameAsBytes()</code>.
229   * @param tablename         - name of table
230   * @param logSeqNum         - log sequence number
231   * @param now               Time at which this edit was written.
232   * @param clusterIds        the clusters that have consumed the change(used in Replication)
233   * @param nonceGroup        the nonceGroup
234   * @param nonce             the nonce
235   * @param mvcc              the mvcc associate the WALKeyImpl
236   * @param replicationScope  the non-default replication scope
237   *                          associated with the region's column families
238   */
239  // TODO: Fix being able to pass in sequenceid.
240  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename, long logSeqNum,
241      final long now, List<UUID> clusterIds, long nonceGroup, long nonce,
242      MultiVersionConcurrencyControl mvcc, final NavigableMap<byte[], Integer> replicationScope) {
243    init(encodedRegionName, tablename, logSeqNum, now, clusterIds, nonceGroup, nonce, mvcc,
244        replicationScope, null);
245  }
246
247  /**
248   * Create the log key for writing to somewhere.
249   * We maintain the tablename mainly for debugging purposes.
250   * A regionName is always a sub-table object.
251   * <p>Used by log splitting and snapshots.
252   *
253   * @param encodedRegionName Encoded name of the region as returned by
254   *                          <code>HRegionInfo#getEncodedNameAsBytes()</code>.
255   * @param tablename         - name of table
256   * @param logSeqNum         - log sequence number
257   * @param now               Time at which this edit was written.
258   * @param clusterIds        the clusters that have consumed the change(used in Replication)
259   */
260  // TODO: Fix being able to pass in sequenceid.
261  public WALKeyImpl(final byte[] encodedRegionName,
262                final TableName tablename,
263                long logSeqNum,
264                final long now,
265                List<UUID> clusterIds,
266                long nonceGroup,
267                long nonce,
268                MultiVersionConcurrencyControl mvcc) {
269    init(encodedRegionName, tablename, logSeqNum, now, clusterIds, nonceGroup,
270        nonce, mvcc, null, null);
271  }
272
273  /**
274   * Create the log key for writing to somewhere.
275   * We maintain the tablename mainly for debugging purposes.
276   * A regionName is always a sub-table object.
277   *
278   * @param encodedRegionName Encoded name of the region as returned by
279   *                          <code>HRegionInfo#getEncodedNameAsBytes()</code>.
280   * @param tablename         the tablename
281   * @param now               Time at which this edit was written.
282   * @param clusterIds        the clusters that have consumed the change(used in Replication)
283   * @param nonceGroup
284   * @param nonce
285   * @param mvcc mvcc control used to generate sequence numbers and control read/write points
286   */
287  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename,
288                final long now, List<UUID> clusterIds, long nonceGroup,
289                final long nonce, final MultiVersionConcurrencyControl mvcc) {
290    init(encodedRegionName, tablename, NO_SEQUENCE_ID, now, clusterIds, nonceGroup, nonce, mvcc,
291        null, null);
292  }
293
294  /**
295   * Create the log key for writing to somewhere.
296   * We maintain the tablename mainly for debugging purposes.
297   * A regionName is always a sub-table object.
298   *
299   * @param encodedRegionName Encoded name of the region as returned by
300   *                          <code>HRegionInfo#getEncodedNameAsBytes()</code>.
301   * @param tablename
302   * @param now               Time at which this edit was written.
303   * @param clusterIds        the clusters that have consumed the change(used in Replication)
304   * @param nonceGroup        the nonceGroup
305   * @param nonce             the nonce
306   * @param mvcc mvcc control used to generate sequence numbers and control read/write points
307   * @param replicationScope  the non-default replication scope of the column families
308   */
309  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename,
310                final long now, List<UUID> clusterIds, long nonceGroup,
311                final long nonce, final MultiVersionConcurrencyControl mvcc,
312                NavigableMap<byte[], Integer> replicationScope) {
313    init(encodedRegionName, tablename, NO_SEQUENCE_ID, now, clusterIds, nonceGroup, nonce, mvcc,
314        replicationScope, null);
315  }
316
317  /**
318   * Create the log key for writing to somewhere.
319   * We maintain the tablename mainly for debugging purposes.
320   * A regionName is always a sub-table object.
321   *
322   * @param encodedRegionName Encoded name of the region as returned by
323   *                          <code>HRegionInfo#getEncodedNameAsBytes()</code>.
324   * @param tablename
325   * @param logSeqNum
326   * @param nonceGroup
327   * @param nonce
328   */
329  // TODO: Fix being able to pass in sequenceid.
330  public WALKeyImpl(final byte[] encodedRegionName,
331                final TableName tablename,
332                long logSeqNum,
333                long nonceGroup,
334                long nonce,
335                final MultiVersionConcurrencyControl mvcc) {
336    init(encodedRegionName,
337        tablename,
338        logSeqNum,
339        EnvironmentEdgeManager.currentTime(),
340        EMPTY_UUIDS,
341        nonceGroup,
342        nonce,
343        mvcc, null, null);
344  }
345
346  public WALKeyImpl(final byte[] encodedRegionName, final TableName tablename,
347                    final long now, List<UUID> clusterIds, long nonceGroup,
348                    final long nonce, final MultiVersionConcurrencyControl mvcc,
349                    NavigableMap<byte[], Integer> replicationScope,
350                    Map<String, byte[]> extendedAttributes){
351    init(encodedRegionName,
352        tablename,
353        NO_SEQUENCE_ID,
354        now,
355        clusterIds,
356        nonceGroup,
357        nonce,
358        mvcc, replicationScope, extendedAttributes);
359  }
360
361  @InterfaceAudience.Private
362  protected void init(final byte[] encodedRegionName,
363                      final TableName tablename,
364                      long logSeqNum,
365                      final long now,
366                      List<UUID> clusterIds,
367                      long nonceGroup,
368                      long nonce,
369                      MultiVersionConcurrencyControl mvcc,
370                      NavigableMap<byte[], Integer> replicationScope,
371                      Map<String, byte[]> extendedAttributes) {
372    this.sequenceId = logSeqNum;
373    this.writeTime = now;
374    this.clusterIds = clusterIds;
375    this.encodedRegionName = encodedRegionName;
376    this.tablename = tablename;
377    this.nonceGroup = nonceGroup;
378    this.nonce = nonce;
379    this.mvcc = mvcc;
380    if (logSeqNum != NO_SEQUENCE_ID) {
381      setSequenceId(logSeqNum);
382    }
383    this.replicationScope = replicationScope;
384    this.extendedAttributes = extendedAttributes;
385  }
386
387  // For deserialization. DO NOT USE. See setWriteEntry below.
388  @InterfaceAudience.Private
389  protected void setSequenceId(long sequenceId) {
390    this.sequenceId = sequenceId;
391  }
392
393  /**
394   * @param compressionContext Compression context to use
395   * @deprecated deparcated since hbase 2.1.0
396   */
397  @Deprecated
398  public void setCompressionContext(CompressionContext compressionContext) {
399    //do nothing
400  }
401
402  /** @return encoded region name */
403  @Override
404  public byte [] getEncodedRegionName() {
405    return encodedRegionName;
406  }
407
408  /** @return table name */
409  @Override
410  public TableName getTableName() {
411    return tablename;
412  }
413
414  /** @return log sequence number
415   * @deprecated Use {@link #getSequenceId()}
416   */
417  @Deprecated
418  public long getLogSeqNum() {
419    return getSequenceId();
420  }
421
422  /**
423   * Used to set original sequenceId for WALKeyImpl during WAL replay
424   */
425  public void setOrigLogSeqNum(final long sequenceId) {
426    this.origLogSeqNum = sequenceId;
427  }
428
429  /**
430   * Return a positive long if current WALKeyImpl is created from a replay edit; a replay edit is an
431   * edit that came in when replaying WALs of a crashed server.
432   * @return original sequence number of the WALEdit
433   */
434  @Override
435  public long getOrigLogSeqNum() {
436    return this.origLogSeqNum;
437  }
438
439  /**
440   * SequenceId is only available post WAL-assign. Calls before this will get you a
441   * {@link SequenceId#NO_SEQUENCE_ID}. See the comment on FSHLog#append and #getWriteNumber in this
442   * method for more on when this sequenceId comes available.
443   * @return long the new assigned sequence number
444   */
445  @Override
446  public long getSequenceId() {
447    return this.sequenceId;
448  }
449
450  /**
451   * @return the write time
452   */
453  @Override
454  public long getWriteTime() {
455    return this.writeTime;
456  }
457
458  public NavigableMap<byte[], Integer> getReplicationScopes() {
459    return replicationScope;
460  }
461
462  /** @return The nonce group */
463  @Override
464  public long getNonceGroup() {
465    return nonceGroup;
466  }
467
468  /** @return The nonce */
469  @Override
470  public long getNonce() {
471    return nonce;
472  }
473
474  private void setReplicationScope(NavigableMap<byte[], Integer> replicationScope) {
475    this.replicationScope = replicationScope;
476  }
477
478  public void clearReplicationScope() {
479    setReplicationScope(null);
480  }
481
482  /**
483   * Marks that the cluster with the given clusterId has consumed the change
484   */
485  public void addClusterId(UUID clusterId) {
486    if (!clusterIds.contains(clusterId)) {
487      clusterIds.add(clusterId);
488    }
489  }
490
491  /**
492   * @return the set of cluster Ids that have consumed the change
493   */
494  public List<UUID> getClusterIds() {
495    return clusterIds;
496  }
497
498  /**
499   * @return the cluster id on which the change has originated. It there is no such cluster, it
500   *         returns DEFAULT_CLUSTER_ID (cases where replication is not enabled)
501   */
502  @Override
503  public UUID getOriginatingClusterId(){
504    return clusterIds.isEmpty()? HConstants.DEFAULT_CLUSTER_ID: clusterIds.get(0);
505  }
506
507  @Override
508  public void addExtendedAttribute(String attributeKey, byte[] attributeValue){
509    if (extendedAttributes == null){
510      extendedAttributes = new HashMap<String, byte[]>();
511    }
512    extendedAttributes.put(attributeKey, attributeValue);
513  }
514
515  @Override
516  public byte[] getExtendedAttribute(String attributeKey){
517    return extendedAttributes != null ? extendedAttributes.get(attributeKey) : null;
518  }
519
520  @Override
521  public Map<String, byte[]> getExtendedAttributes(){
522    return extendedAttributes != null ? new HashMap<String, byte[]>(extendedAttributes) :
523        new HashMap<String, byte[]>();
524  }
525
526  @Override
527  public String toString() {
528    return tablename + "/" + Bytes.toString(encodedRegionName) + "/" + sequenceId;
529  }
530
531  @Override
532  public boolean equals(Object obj) {
533    if (this == obj) {
534      return true;
535    }
536    if (obj == null || getClass() != obj.getClass()) {
537      return false;
538    }
539    return compareTo((WALKey)obj) == 0;
540  }
541
542  @Override
543  public int hashCode() {
544    int result = Bytes.hashCode(this.encodedRegionName);
545    result = (int) (result ^ getSequenceId());
546    result = (int) (result ^ this.writeTime);
547    return result;
548  }
549
550  @Override
551  public int compareTo(WALKey o) {
552    int result = Bytes.compareTo(this.encodedRegionName, o.getEncodedRegionName());
553    if (result == 0) {
554      long sid = getSequenceId();
555      long otherSid = o.getSequenceId();
556      if (sid < otherSid) {
557        result = -1;
558      } else if (sid  > otherSid) {
559        result = 1;
560      }
561      if (result == 0) {
562        if (this.writeTime < o.getWriteTime()) {
563          result = -1;
564        } else if (this.writeTime > o.getWriteTime()) {
565          return 1;
566        }
567      }
568    }
569    // why isn't cluster id accounted for?
570    return result;
571  }
572
573  /**
574   * Drop this instance's tablename byte array and instead
575   * hold a reference to the provided tablename. This is not
576   * meant to be a general purpose setter - it's only used
577   * to collapse references to conserve memory.
578   */
579  void internTableName(TableName tablename) {
580    // We should not use this as a setter - only to swap
581    // in a new reference to the same table name.
582    assert tablename.equals(this.tablename);
583    this.tablename = tablename;
584  }
585
586  /**
587   * Drop this instance's region name byte array and instead
588   * hold a reference to the provided region name. This is not
589   * meant to be a general purpose setter - it's only used
590   * to collapse references to conserve memory.
591   */
592  void internEncodedRegionName(byte []encodedRegionName) {
593    // We should not use this as a setter - only to swap
594    // in a new reference to the same table name.
595    assert Bytes.equals(this.encodedRegionName, encodedRegionName);
596    this.encodedRegionName = encodedRegionName;
597  }
598
599  public WALProtos.WALKey.Builder getBuilder(WALCellCodec.ByteStringCompressor compressor)
600      throws IOException {
601    WALProtos.WALKey.Builder builder = WALProtos.WALKey.newBuilder();
602    builder.setEncodedRegionName(
603      compressor.compress(this.encodedRegionName, CompressionContext.DictionaryIndex.REGION));
604    builder.setTableName(
605      compressor.compress(this.tablename.getName(), CompressionContext.DictionaryIndex.TABLE));
606    builder.setLogSequenceNumber(getSequenceId());
607    builder.setWriteTime(writeTime);
608    if (this.origLogSeqNum > 0) {
609      builder.setOrigSequenceNumber(this.origLogSeqNum);
610    }
611    if (this.nonce != HConstants.NO_NONCE) {
612      builder.setNonce(nonce);
613    }
614    if (this.nonceGroup != HConstants.NO_NONCE) {
615      builder.setNonceGroup(nonceGroup);
616    }
617    HBaseProtos.UUID.Builder uuidBuilder = HBaseProtos.UUID.newBuilder();
618    for (UUID clusterId : clusterIds) {
619      uuidBuilder.setLeastSigBits(clusterId.getLeastSignificantBits());
620      uuidBuilder.setMostSigBits(clusterId.getMostSignificantBits());
621      builder.addClusterIds(uuidBuilder.build());
622    }
623    if (replicationScope != null) {
624      for (Map.Entry<byte[], Integer> e : replicationScope.entrySet()) {
625        ByteString family =
626            compressor.compress(e.getKey(), CompressionContext.DictionaryIndex.FAMILY);
627        builder.addScopes(FamilyScope.newBuilder().setFamily(family)
628            .setScopeType(ScopeType.forNumber(e.getValue())));
629      }
630    }
631    if (extendedAttributes != null){
632      for (Map.Entry<String, byte[]> e : extendedAttributes.entrySet()){
633        WALProtos.Attribute attr = WALProtos.Attribute.newBuilder().
634            setKey(e.getKey()).setValue(compressor.compress(e.getValue(),
635            CompressionContext.DictionaryIndex.TABLE)).build();
636        builder.addExtendedAttributes(attr);
637      }
638    }
639    return builder;
640  }
641
642  public void readFieldsFromPb(WALProtos.WALKey walKey,
643      WALCellCodec.ByteStringUncompressor uncompressor) throws IOException {
644    this.encodedRegionName = uncompressor.uncompress(walKey.getEncodedRegionName(),
645      CompressionContext.DictionaryIndex.REGION);
646    byte[] tablenameBytes =
647        uncompressor.uncompress(walKey.getTableName(), CompressionContext.DictionaryIndex.TABLE);
648    this.tablename = TableName.valueOf(tablenameBytes);
649    clusterIds.clear();
650    for (HBaseProtos.UUID clusterId : walKey.getClusterIdsList()) {
651      clusterIds.add(new UUID(clusterId.getMostSigBits(), clusterId.getLeastSigBits()));
652    }
653    if (walKey.hasNonceGroup()) {
654      this.nonceGroup = walKey.getNonceGroup();
655    }
656    if (walKey.hasNonce()) {
657      this.nonce = walKey.getNonce();
658    }
659    this.replicationScope = null;
660    if (walKey.getScopesCount() > 0) {
661      this.replicationScope = new TreeMap<>(Bytes.BYTES_COMPARATOR);
662      for (FamilyScope scope : walKey.getScopesList()) {
663        byte[] family =
664            uncompressor.uncompress(scope.getFamily(), CompressionContext.DictionaryIndex.FAMILY);
665        this.replicationScope.put(family, scope.getScopeType().getNumber());
666      }
667    }
668    setSequenceId(walKey.getLogSequenceNumber());
669    this.writeTime = walKey.getWriteTime();
670    if (walKey.hasOrigSequenceNumber()) {
671      this.origLogSeqNum = walKey.getOrigSequenceNumber();
672    }
673    if (walKey.getExtendedAttributesCount() > 0){
674      this.extendedAttributes = new HashMap<>(walKey.getExtendedAttributesCount());
675      for (WALProtos.Attribute attr : walKey.getExtendedAttributesList()){
676        byte[] value =
677            uncompressor.uncompress(attr.getValue(), CompressionContext.DictionaryIndex.TABLE);
678        extendedAttributes.put(attr.getKey(), value);
679      }
680    }
681  }
682
683  @Override
684  public long estimatedSerializedSizeOf() {
685    long size = encodedRegionName != null ? encodedRegionName.length : 0;
686    size += tablename != null ? tablename.toBytes().length : 0;
687    if (clusterIds != null) {
688      size += 16 * clusterIds.size();
689    }
690    if (nonceGroup != HConstants.NO_NONCE) {
691      size += Bytes.SIZEOF_LONG; // nonce group
692    }
693    if (nonce != HConstants.NO_NONCE) {
694      size += Bytes.SIZEOF_LONG; // nonce
695    }
696    if (replicationScope != null) {
697      for (Map.Entry<byte[], Integer> scope: replicationScope.entrySet()) {
698        size += scope.getKey().length;
699        size += Bytes.SIZEOF_INT;
700      }
701    }
702    size += Bytes.SIZEOF_LONG; // sequence number
703    size += Bytes.SIZEOF_LONG; // write time
704    if (origLogSeqNum > 0) {
705      size += Bytes.SIZEOF_LONG; // original sequence number
706    }
707    return size;
708  }
709}