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.io.hfile;
019
020import java.io.ByteArrayInputStream;
021import java.io.ByteArrayOutputStream;
022import java.io.DataInput;
023import java.io.DataInputStream;
024import java.io.DataOutputStream;
025import java.io.IOException;
026import java.nio.ByteBuffer;
027import org.apache.hadoop.fs.FSDataInputStream;
028import org.apache.hadoop.hbase.CellComparator;
029import org.apache.hadoop.hbase.CellComparatorImpl;
030import org.apache.hadoop.hbase.KeyValue;
031import org.apache.hadoop.hbase.MetaCellComparator;
032import org.apache.hadoop.hbase.io.compress.Compression;
033import org.apache.hadoop.hbase.util.Bytes;
034import org.apache.yetus.audience.InterfaceAudience;
035import org.slf4j.Logger;
036import org.slf4j.LoggerFactory;
037
038import org.apache.hbase.thirdparty.com.google.protobuf.UnsafeByteOperations;
039
040import org.apache.hadoop.hbase.shaded.protobuf.generated.HFileProtos;
041
042/**
043 * The {@link HFile} has a fixed trailer which contains offsets to other variable parts of the file.
044 * Also includes basic metadata on this file. The trailer size is fixed within a given {@link HFile}
045 * format version only, but we always store the version number as the last four-byte integer of the
046 * file. The version number itself is split into two portions, a major version and a minor version.
047 * The last three bytes of a file are the major version and a single preceding byte is the minor
048 * number. The major version determines which readers/writers to use to read/write a hfile while a
049 * minor version determines smaller changes in hfile format that do not need a new reader/writer
050 * type.
051 */
052@InterfaceAudience.Private
053public class FixedFileTrailer {
054  private static final Logger LOG = LoggerFactory.getLogger(FixedFileTrailer.class);
055
056  /**
057   * We store the comparator class name as a fixed-length field in the trailer.
058   */
059  private static final int MAX_COMPARATOR_NAME_LENGTH = 128;
060
061  /**
062   * Offset to the fileinfo data, a small block of vitals. Necessary in v1 but only potentially
063   * useful for pretty-printing in v2.
064   */
065  private long fileInfoOffset;
066
067  /**
068   * In version 1, the offset to the data block index. Starting from version 2, the meaning of this
069   * field is the offset to the section of the file that should be loaded at the time the file is
070   * being opened: i.e. on open we load the root index, file info, etc. See
071   * http://hbase.apache.org/book.html#_hfile_format_2 in the reference guide.
072   */
073  private long loadOnOpenDataOffset;
074
075  /**
076   * The number of entries in the root data index.
077   */
078  private int dataIndexCount;
079
080  /**
081   * Total uncompressed size of all blocks of the data index
082   */
083  private long uncompressedDataIndexSize;
084
085  /**
086   * The number of entries in the meta index
087   */
088  private int metaIndexCount;
089
090  /**
091   * The total uncompressed size of keys/values stored in the file.
092   */
093  private long totalUncompressedBytes;
094
095  /**
096   * The number of key/value pairs in the file. This field was int in version 1, but is now long.
097   */
098  private long entryCount;
099
100  /**
101   * The compression codec used for all blocks.
102   */
103  private Compression.Algorithm compressionCodec = Compression.Algorithm.NONE;
104
105  /**
106   * The number of levels in the potentially multi-level data index. Used from version 2 onwards.
107   */
108  private int numDataIndexLevels;
109
110  /**
111   * The offset of the first data block.
112   */
113  private long firstDataBlockOffset;
114
115  /**
116   * It is guaranteed that no key/value data blocks start after this offset in the file.
117   */
118  private long lastDataBlockOffset;
119
120  /**
121   * Raw key comparator class name in version 3
122   */
123  // We could write the actual class name from 2.0 onwards and handle BC
124  private String comparatorClassName = CellComparator.getInstance().getClass().getName();
125
126  /**
127   * The encryption key
128   */
129  private byte[] encryptionKey;
130
131  /**
132   * The {@link HFile} format major version.
133   */
134  private final int majorVersion;
135
136  /**
137   * The {@link HFile} format minor version.
138   */
139  private final int minorVersion;
140
141  FixedFileTrailer(int majorVersion, int minorVersion) {
142    this.majorVersion = majorVersion;
143    this.minorVersion = minorVersion;
144    HFile.checkFormatVersion(majorVersion);
145  }
146
147  private static int[] computeTrailerSizeByVersion() {
148    int[] versionToSize = new int[HFile.MAX_FORMAT_VERSION + 1];
149    // We support only 2 major versions now. ie. V2, V3
150    versionToSize[2] = 212;
151    for (int version = 3; version <= HFile.MAX_FORMAT_VERSION; version++) {
152      // Max FFT size for V3 and above is taken as 4KB for future enhancements
153      // if any.
154      // Unless the trailer size exceeds 4K this can continue
155      versionToSize[version] = 1024 * 4;
156    }
157    return versionToSize;
158  }
159
160  private static int getMaxTrailerSize() {
161    int maxSize = 0;
162    for (int version = HFile.MIN_FORMAT_VERSION; version <= HFile.MAX_FORMAT_VERSION; ++version) {
163      maxSize = Math.max(getTrailerSize(version), maxSize);
164    }
165    return maxSize;
166  }
167
168  private static final int[] TRAILER_SIZE = computeTrailerSizeByVersion();
169  private static final int MAX_TRAILER_SIZE = getMaxTrailerSize();
170
171  private static final int NOT_PB_SIZE = BlockType.MAGIC_LENGTH + Bytes.SIZEOF_INT;
172
173  static int getTrailerSize(int version) {
174    return TRAILER_SIZE[version];
175  }
176
177  public int getTrailerSize() {
178    return getTrailerSize(majorVersion);
179  }
180
181  /**
182   * Write the trailer to a data stream. We support writing version 1 for testing and for
183   * determining version 1 trailer size. It is also easy to see what fields changed in version 2.
184   */
185  void serialize(DataOutputStream outputStream) throws IOException {
186    HFile.checkFormatVersion(majorVersion);
187
188    ByteArrayOutputStream baos = new ByteArrayOutputStream();
189    DataOutputStream baosDos = new DataOutputStream(baos);
190
191    BlockType.TRAILER.write(baosDos);
192    serializeAsPB(baosDos);
193
194    // The last 4 bytes of the file encode the major and minor version universally
195    baosDos.writeInt(materializeVersion(majorVersion, minorVersion));
196
197    baos.writeTo(outputStream);
198  }
199
200  HFileProtos.FileTrailerProto toProtobuf() {
201    HFileProtos.FileTrailerProto.Builder builder = HFileProtos.FileTrailerProto.newBuilder()
202      .setFileInfoOffset(fileInfoOffset).setLoadOnOpenDataOffset(loadOnOpenDataOffset)
203      .setUncompressedDataIndexSize(uncompressedDataIndexSize)
204      .setTotalUncompressedBytes(totalUncompressedBytes).setDataIndexCount(dataIndexCount)
205      .setMetaIndexCount(metaIndexCount).setEntryCount(entryCount)
206      .setNumDataIndexLevels(numDataIndexLevels).setFirstDataBlockOffset(firstDataBlockOffset)
207      .setLastDataBlockOffset(lastDataBlockOffset)
208      .setComparatorClassName(getHBase1CompatibleName(comparatorClassName))
209      .setCompressionCodec(compressionCodec.ordinal());
210    if (encryptionKey != null) {
211      builder.setEncryptionKey(UnsafeByteOperations.unsafeWrap(encryptionKey));
212    }
213    return builder.build();
214  }
215
216  /**
217   * Write trailer data as protobuf. NOTE: we run a translation on the comparator name and will
218   * serialize the old hbase-1.x where it makes sense. See {@link #getHBase1CompatibleName(String)}.
219   */
220  void serializeAsPB(DataOutputStream output) throws IOException {
221    ByteArrayOutputStream baos = new ByteArrayOutputStream();
222    // We need this extra copy unfortunately to determine the final size of the
223    // delimited output, see use of baos.size() below.
224    toProtobuf().writeDelimitedTo(baos);
225    baos.writeTo(output);
226    // Pad to make up the difference between variable PB encoding length and the
227    // length when encoded as writable under earlier V2 formats. Failure to pad
228    // properly or if the PB encoding is too big would mean the trailer wont be read
229    // in properly by HFile.
230    int padding = getTrailerSize() - NOT_PB_SIZE - baos.size();
231    if (padding < 0) {
232      throw new IOException("Pbuf encoding size exceeded fixed trailer size limit");
233    }
234    for (int i = 0; i < padding; i++) {
235      output.write(0);
236    }
237  }
238
239  /**
240   * Deserialize the fixed file trailer from the given stream. The version needs to already be
241   * specified. Make sure this is consistent with {@link #serialize(DataOutputStream)}.
242   */
243  void deserialize(DataInputStream inputStream) throws IOException {
244    HFile.checkFormatVersion(majorVersion);
245
246    BlockType.TRAILER.readAndCheck(inputStream);
247
248    if (
249      majorVersion > 2
250        || (majorVersion == 2 && minorVersion >= HFileReaderImpl.PBUF_TRAILER_MINOR_VERSION)
251    ) {
252      deserializeFromPB(inputStream);
253    } else {
254      deserializeFromWritable(inputStream);
255    }
256
257    // The last 4 bytes of the file encode the major and minor version universally
258    int version = inputStream.readInt();
259    expectMajorVersion(extractMajorVersion(version));
260    expectMinorVersion(extractMinorVersion(version));
261  }
262
263  /**
264   * Deserialize the file trailer as protobuf
265   */
266  void deserializeFromPB(DataInputStream inputStream) throws IOException {
267    // read PB and skip padding
268    int start = inputStream.available();
269    HFileProtos.FileTrailerProto trailerProto =
270      HFileProtos.FileTrailerProto.PARSER.parseDelimitedFrom(inputStream);
271    int size = start - inputStream.available();
272    inputStream.skip(getTrailerSize() - NOT_PB_SIZE - size);
273
274    // process the PB
275    if (trailerProto.hasFileInfoOffset()) {
276      fileInfoOffset = trailerProto.getFileInfoOffset();
277    }
278    if (trailerProto.hasLoadOnOpenDataOffset()) {
279      loadOnOpenDataOffset = trailerProto.getLoadOnOpenDataOffset();
280    }
281    if (trailerProto.hasUncompressedDataIndexSize()) {
282      uncompressedDataIndexSize = trailerProto.getUncompressedDataIndexSize();
283    }
284    if (trailerProto.hasTotalUncompressedBytes()) {
285      totalUncompressedBytes = trailerProto.getTotalUncompressedBytes();
286    }
287    if (trailerProto.hasDataIndexCount()) {
288      dataIndexCount = trailerProto.getDataIndexCount();
289    }
290    if (trailerProto.hasMetaIndexCount()) {
291      metaIndexCount = trailerProto.getMetaIndexCount();
292    }
293    if (trailerProto.hasEntryCount()) {
294      entryCount = trailerProto.getEntryCount();
295    }
296    if (trailerProto.hasNumDataIndexLevels()) {
297      numDataIndexLevels = trailerProto.getNumDataIndexLevels();
298    }
299    if (trailerProto.hasFirstDataBlockOffset()) {
300      firstDataBlockOffset = trailerProto.getFirstDataBlockOffset();
301    }
302    if (trailerProto.hasLastDataBlockOffset()) {
303      lastDataBlockOffset = trailerProto.getLastDataBlockOffset();
304    }
305    if (trailerProto.hasComparatorClassName()) {
306      setComparatorClass(getComparatorClass(trailerProto.getComparatorClassName()));
307    }
308    if (trailerProto.hasCompressionCodec()) {
309      compressionCodec = Compression.Algorithm.values()[trailerProto.getCompressionCodec()];
310    } else {
311      compressionCodec = Compression.Algorithm.NONE;
312    }
313    if (trailerProto.hasEncryptionKey()) {
314      encryptionKey = trailerProto.getEncryptionKey().toByteArray();
315    }
316  }
317
318  /**
319   * Deserialize the file trailer as writable data
320   */
321  void deserializeFromWritable(DataInput input) throws IOException {
322    fileInfoOffset = input.readLong();
323    loadOnOpenDataOffset = input.readLong();
324    dataIndexCount = input.readInt();
325    uncompressedDataIndexSize = input.readLong();
326    metaIndexCount = input.readInt();
327
328    totalUncompressedBytes = input.readLong();
329    entryCount = input.readLong();
330    compressionCodec = Compression.Algorithm.values()[input.readInt()];
331    numDataIndexLevels = input.readInt();
332    firstDataBlockOffset = input.readLong();
333    lastDataBlockOffset = input.readLong();
334    // TODO this is a classname encoded into an HFile's trailer. We are going to need to have
335    // some compat code here.
336    setComparatorClass(
337      getComparatorClass(Bytes.readStringFixedSize(input, MAX_COMPARATOR_NAME_LENGTH)));
338  }
339
340  private void append(StringBuilder sb, String s) {
341    if (sb.length() > 0) {
342      sb.append(", ");
343    }
344    sb.append(s);
345  }
346
347  @Override
348  public String toString() {
349    StringBuilder sb = new StringBuilder();
350    append(sb, "fileinfoOffset=" + fileInfoOffset);
351    append(sb, "loadOnOpenDataOffset=" + loadOnOpenDataOffset);
352    append(sb, "dataIndexCount=" + dataIndexCount);
353    append(sb, "metaIndexCount=" + metaIndexCount);
354    append(sb, "totalUncomressedBytes=" + totalUncompressedBytes);
355    append(sb, "entryCount=" + entryCount);
356    append(sb, "compressionCodec=" + compressionCodec);
357    append(sb, "uncompressedDataIndexSize=" + uncompressedDataIndexSize);
358    append(sb, "numDataIndexLevels=" + numDataIndexLevels);
359    append(sb, "firstDataBlockOffset=" + firstDataBlockOffset);
360    append(sb, "lastDataBlockOffset=" + lastDataBlockOffset);
361    append(sb, "comparatorClassName=" + comparatorClassName);
362    if (majorVersion >= 3) {
363      append(sb, "encryptionKey=" + (encryptionKey != null ? "PRESENT" : "NONE"));
364    }
365    append(sb, "majorVersion=" + majorVersion);
366    append(sb, "minorVersion=" + minorVersion);
367
368    return sb.toString();
369  }
370
371  /**
372   * Reads a file trailer from the given file.
373   * @param istream  the input stream with the ability to seek. Does not have to be buffered, as
374   *                 only one read operation is made.
375   * @param fileSize the file size. Can be obtained using
376   *                 {@link org.apache.hadoop.fs.FileSystem#getFileStatus( org.apache.hadoop.fs.Path)}.
377   * @return the fixed file trailer read
378   * @throws IOException if failed to read from the underlying stream, or the trailer is corrupted,
379   *                     or the version of the trailer is unsupported
380   */
381  public static FixedFileTrailer readFromStream(FSDataInputStream istream, long fileSize)
382    throws IOException {
383    int bufferSize = MAX_TRAILER_SIZE;
384    long seekPoint = fileSize - bufferSize;
385    if (seekPoint < 0) {
386      // It is hard to imagine such a small HFile.
387      seekPoint = 0;
388      bufferSize = (int) fileSize;
389    }
390
391    HFileUtil.seekOnMultipleSources(istream, seekPoint);
392
393    ByteBuffer buf = ByteBuffer.allocate(bufferSize);
394    istream.readFully(buf.array(), buf.arrayOffset(), buf.arrayOffset() + buf.limit());
395
396    // Read the version from the last int of the file.
397    buf.position(buf.limit() - Bytes.SIZEOF_INT);
398    int version = buf.getInt();
399
400    // Extract the major and minor versions.
401    int majorVersion = extractMajorVersion(version);
402    int minorVersion = extractMinorVersion(version);
403
404    HFile.checkFormatVersion(majorVersion); // throws IAE if invalid
405
406    int trailerSize = getTrailerSize(majorVersion);
407
408    FixedFileTrailer fft = new FixedFileTrailer(majorVersion, minorVersion);
409    fft.deserialize(new DataInputStream(new ByteArrayInputStream(buf.array(),
410      buf.arrayOffset() + bufferSize - trailerSize, trailerSize)));
411    return fft;
412  }
413
414  public void expectMajorVersion(int expected) {
415    if (majorVersion != expected) {
416      throw new IllegalArgumentException(
417        "Invalid HFile major version: " + majorVersion + " (expected: " + expected + ")");
418    }
419  }
420
421  public void expectMinorVersion(int expected) {
422    if (minorVersion != expected) {
423      throw new IllegalArgumentException(
424        "Invalid HFile minor version: " + minorVersion + " (expected: " + expected + ")");
425    }
426  }
427
428  public void expectAtLeastMajorVersion(int lowerBound) {
429    if (majorVersion < lowerBound) {
430      throw new IllegalArgumentException("Invalid HFile major version: " + majorVersion
431        + " (expected: " + lowerBound + " or higher).");
432    }
433  }
434
435  public long getFileInfoOffset() {
436    return fileInfoOffset;
437  }
438
439  public void setFileInfoOffset(long fileInfoOffset) {
440    this.fileInfoOffset = fileInfoOffset;
441  }
442
443  public long getLoadOnOpenDataOffset() {
444    return loadOnOpenDataOffset;
445  }
446
447  public void setLoadOnOpenOffset(long loadOnOpenDataOffset) {
448    this.loadOnOpenDataOffset = loadOnOpenDataOffset;
449  }
450
451  public int getDataIndexCount() {
452    return dataIndexCount;
453  }
454
455  public void setDataIndexCount(int dataIndexCount) {
456    this.dataIndexCount = dataIndexCount;
457  }
458
459  public int getMetaIndexCount() {
460    return metaIndexCount;
461  }
462
463  public void setMetaIndexCount(int metaIndexCount) {
464    this.metaIndexCount = metaIndexCount;
465  }
466
467  public long getTotalUncompressedBytes() {
468    return totalUncompressedBytes;
469  }
470
471  public void setTotalUncompressedBytes(long totalUncompressedBytes) {
472    this.totalUncompressedBytes = totalUncompressedBytes;
473  }
474
475  public long getEntryCount() {
476    return entryCount;
477  }
478
479  public void setEntryCount(long newEntryCount) {
480    entryCount = newEntryCount;
481  }
482
483  public Compression.Algorithm getCompressionCodec() {
484    return compressionCodec;
485  }
486
487  public void setCompressionCodec(Compression.Algorithm compressionCodec) {
488    this.compressionCodec = compressionCodec;
489  }
490
491  public int getNumDataIndexLevels() {
492    expectAtLeastMajorVersion(2);
493    return numDataIndexLevels;
494  }
495
496  public void setNumDataIndexLevels(int numDataIndexLevels) {
497    expectAtLeastMajorVersion(2);
498    this.numDataIndexLevels = numDataIndexLevels;
499  }
500
501  public long getLastDataBlockOffset() {
502    expectAtLeastMajorVersion(2);
503    return lastDataBlockOffset;
504  }
505
506  public void setLastDataBlockOffset(long lastDataBlockOffset) {
507    expectAtLeastMajorVersion(2);
508    this.lastDataBlockOffset = lastDataBlockOffset;
509  }
510
511  public long getFirstDataBlockOffset() {
512    expectAtLeastMajorVersion(2);
513    return firstDataBlockOffset;
514  }
515
516  public void setFirstDataBlockOffset(long firstDataBlockOffset) {
517    expectAtLeastMajorVersion(2);
518    this.firstDataBlockOffset = firstDataBlockOffset;
519  }
520
521  public String getComparatorClassName() {
522    return comparatorClassName;
523  }
524
525  /**
526   * Returns the major version of this HFile format
527   */
528  public int getMajorVersion() {
529    return majorVersion;
530  }
531
532  /**
533   * Returns the minor version of this HFile format
534   */
535  public int getMinorVersion() {
536    return minorVersion;
537  }
538
539  public void setComparatorClass(Class<? extends CellComparator> klass) {
540    // Is the comparator instantiable?
541    try {
542      // If null, it should be the Bytes.BYTES_RAWCOMPARATOR
543      if (klass != null) {
544        CellComparator comp = klass.getDeclaredConstructor().newInstance();
545        // if the name wasn't one of the legacy names, maybe its a legit new
546        // kind of comparator.
547        this.comparatorClassName = klass.getName();
548      }
549    } catch (Exception e) {
550      throw new RuntimeException("Comparator class " + klass.getName() + " is not instantiable", e);
551    }
552  }
553
554  /**
555   * If a 'standard' Comparator, write the old name for the Comparator when we serialize rather than
556   * the new name; writing the new name will make it so newly-written hfiles are not parseable by
557   * hbase-1.x, a facility we'd like to preserve across rolling upgrade and hbase-1.x clusters
558   * reading hbase-2.x produce.
559   * <p>
560   * The Comparators in hbase-2.x work the same as they did in hbase-1.x; they compare KeyValues. In
561   * hbase-2.x they were renamed making use of the more generic 'Cell' nomenclature to indicate that
562   * we intend to move away from KeyValues post hbase-2. A naming change is not reason enough to
563   * make it so hbase-1.x cannot read hbase-2.x files given the structure goes unchanged (hfile v3).
564   * So, lets write the old names for Comparators into the hfile tails in hbase-2. Here is where we
565   * do the translation. {@link #getComparatorClass(String)} does translation going the other way.
566   * <p>
567   * The translation is done on the serialized Protobuf only.
568   * </p>
569   * @param comparator String class name of the Comparator used in this hfile.
570   * @return What to store in the trailer as our comparator name.
571   * @see #getComparatorClass(String)
572   * @since hbase-2.0.0.
573   * @deprecated Since hbase-2.0.0. Will be removed in hbase-3.0.0.
574   */
575  @Deprecated
576  private String getHBase1CompatibleName(final String comparator) {
577    if (comparator.equals(CellComparatorImpl.class.getName())) {
578      return KeyValue.COMPARATOR.getClass().getName();
579    }
580    if (comparator.equals(MetaCellComparator.class.getName())) {
581      return KeyValue.META_COMPARATOR.getClass().getName();
582    }
583    return comparator;
584  }
585
586  @SuppressWarnings("unchecked")
587  private static Class<? extends CellComparator> getComparatorClass(String comparatorClassName)
588    throws IOException {
589    Class<? extends CellComparator> comparatorKlass;
590    // for BC
591    if (
592      comparatorClassName.equals(KeyValue.COMPARATOR.getLegacyKeyComparatorName())
593        || comparatorClassName.equals(KeyValue.COMPARATOR.getClass().getName())
594        || (comparatorClassName.equals("org.apache.hadoop.hbase.CellComparator"))
595    ) {
596      comparatorKlass = CellComparatorImpl.class;
597    } else if (
598      comparatorClassName.equals(KeyValue.META_COMPARATOR.getLegacyKeyComparatorName())
599        || comparatorClassName.equals(KeyValue.META_COMPARATOR.getClass().getName())
600        || (comparatorClassName.equals("org.apache.hadoop.hbase.CellComparator$MetaCellComparator"))
601        || (comparatorClassName
602          .equals("org.apache.hadoop.hbase.CellComparatorImpl$MetaCellComparator"))
603        || (comparatorClassName.equals("org.apache.hadoop.hbase.MetaCellComparator"))
604    ) {
605      comparatorKlass = MetaCellComparator.class;
606    } else if (
607      comparatorClassName.equals("org.apache.hadoop.hbase.KeyValue$RawBytesComparator")
608        || comparatorClassName.equals("org.apache.hadoop.hbase.util.Bytes$ByteArrayComparator")
609    ) {
610      // When the comparator to be used is Bytes.BYTES_RAWCOMPARATOR, we just return null from here
611      // Bytes.BYTES_RAWCOMPARATOR is not a CellComparator
612      comparatorKlass = null;
613    } else {
614      // if the name wasn't one of the legacy names, maybe its a legit new kind of comparator.
615      try {
616        comparatorKlass = (Class<? extends CellComparator>) Class.forName(comparatorClassName);
617      } catch (ClassNotFoundException e) {
618        throw new IOException(e);
619      }
620    }
621    return comparatorKlass;
622  }
623
624  static CellComparator createComparator(String comparatorClassName) throws IOException {
625    if (comparatorClassName.equals(CellComparatorImpl.COMPARATOR.getClass().getName())) {
626      return CellComparatorImpl.COMPARATOR;
627    } else
628      if (comparatorClassName.equals(MetaCellComparator.META_COMPARATOR.getClass().getName())) {
629        return MetaCellComparator.META_COMPARATOR;
630      }
631    try {
632      Class<? extends CellComparator> comparatorClass = getComparatorClass(comparatorClassName);
633      if (comparatorClass != null) {
634        return comparatorClass.getDeclaredConstructor().newInstance();
635      }
636      LOG.warn("No Comparator class for " + comparatorClassName + ". Returning Null.");
637      return null;
638    } catch (Exception e) {
639      throw new IOException("Comparator class " + comparatorClassName + " is not instantiable", e);
640    }
641  }
642
643  CellComparator createComparator() throws IOException {
644    expectAtLeastMajorVersion(2);
645    return createComparator(comparatorClassName);
646  }
647
648  public long getUncompressedDataIndexSize() {
649    return uncompressedDataIndexSize;
650  }
651
652  public void setUncompressedDataIndexSize(long uncompressedDataIndexSize) {
653    expectAtLeastMajorVersion(2);
654    this.uncompressedDataIndexSize = uncompressedDataIndexSize;
655  }
656
657  public byte[] getEncryptionKey() {
658    // This is a v3 feature but if reading a v2 file the encryptionKey will just be null which
659    // if fine for this feature.
660    expectAtLeastMajorVersion(2);
661    return encryptionKey;
662  }
663
664  public void setEncryptionKey(byte[] keyBytes) {
665    this.encryptionKey = keyBytes;
666  }
667
668  /**
669   * Extracts the major version for a 4-byte serialized version data. The major version is the 3
670   * least significant bytes
671   */
672  private static int extractMajorVersion(int serializedVersion) {
673    return (serializedVersion & 0x00ffffff);
674  }
675
676  /**
677   * Extracts the minor version for a 4-byte serialized version data. The major version are the 3
678   * the most significant bytes
679   */
680  private static int extractMinorVersion(int serializedVersion) {
681    return (serializedVersion >>> 24);
682  }
683
684  /**
685   * Create a 4 byte serialized version number by combining the minor and major version numbers.
686   */
687  static int materializeVersion(int majorVersion, int minorVersion) {
688    return ((majorVersion & 0x00ffffff) | (minorVersion << 24));
689  }
690}