View Javadoc

1   /*
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  package org.apache.hadoop.hbase.io.hfile;
19  
20  import java.io.IOException;
21  import java.security.Key;
22  import java.security.KeyException;
23  
24  import org.apache.commons.logging.Log;
25  import org.apache.commons.logging.LogFactory;
26  import org.apache.hadoop.hbase.classification.InterfaceAudience;
27  import org.apache.hadoop.conf.Configuration;
28  import org.apache.hadoop.fs.Path;
29  import org.apache.hadoop.hbase.Cell;
30  import org.apache.hadoop.hbase.CellUtil;
31  import org.apache.hadoop.hbase.HConstants;
32  import org.apache.hadoop.hbase.KeyValue;
33  import org.apache.hadoop.hbase.NoTagsKeyValue;
34  import org.apache.hadoop.hbase.fs.HFileSystem;
35  import org.apache.hadoop.hbase.io.FSDataInputStreamWrapper;
36  import org.apache.hadoop.hbase.io.crypto.Cipher;
37  import org.apache.hadoop.hbase.io.crypto.Encryption;
38  import org.apache.hadoop.hbase.io.hfile.HFile.FileInfo;
39  import org.apache.hadoop.hbase.security.EncryptionUtil;
40  import org.apache.hadoop.hbase.security.User;
41  import org.apache.hadoop.hbase.util.ByteBufferUtils;
42  import org.apache.hadoop.hbase.util.Bytes;
43  import org.apache.hadoop.io.WritableUtils;
44  
45  /**
46   * {@link HFile} reader for version 3.
47   */
48  @InterfaceAudience.Private
49  public class HFileReaderV3 extends HFileReaderV2 {
50  
51    private static final Log LOG = LogFactory.getLog(HFileReaderV3.class);
52  
53    public static final int MAX_MINOR_VERSION = 0;
54  
55    /**
56     * Opens a HFile. You must load the index before you can use it by calling
57     * {@link #loadFileInfo()}.
58     * @param path
59     *          Path to HFile.
60     * @param trailer
61     *          File trailer.
62     * @param fsdis
63     *          input stream.
64     * @param size
65     *          Length of the stream.
66     * @param cacheConf
67     *          Cache configuration.
68     * @param hfs
69     *          The file system.
70     * @param conf
71     *          Configuration
72     */
73    public HFileReaderV3(final Path path, FixedFileTrailer trailer,
74        final FSDataInputStreamWrapper fsdis,
75        final long size, final CacheConfig cacheConf, final HFileSystem hfs,
76        final Configuration conf) throws IOException {
77      super(path, trailer, fsdis, size, cacheConf, hfs, conf);
78      byte[] tmp = fileInfo.get(FileInfo.MAX_TAGS_LEN);
79      // max tag length is not present in the HFile means tags were not at all written to file.
80      if (tmp != null) {
81        hfileContext.setIncludesTags(true);
82        tmp = fileInfo.get(FileInfo.TAGS_COMPRESSED);
83        if (tmp != null && Bytes.toBoolean(tmp)) {
84          hfileContext.setCompressTags(true);
85        }
86      }
87    }
88  
89    @Override
90    protected HFileContext createHFileContext(FSDataInputStreamWrapper fsdis, long fileSize,
91        HFileSystem hfs, Path path, FixedFileTrailer trailer) throws IOException {
92      trailer.expectMajorVersion(3);
93      HFileContextBuilder builder = new HFileContextBuilder()
94        .withIncludesMvcc(shouldIncludeMemstoreTS())
95        .withHBaseCheckSum(true)
96        .withCompression(this.compressAlgo);
97  
98      // Check for any key material available
99      byte[] keyBytes = trailer.getEncryptionKey();
100     if (keyBytes != null) {
101       Encryption.Context cryptoContext = Encryption.newContext(conf);
102       Key key;
103       String masterKeyName = conf.get(HConstants.CRYPTO_MASTERKEY_NAME_CONF_KEY,
104         User.getCurrent().getShortName());
105       try {
106         // First try the master key
107         key = EncryptionUtil.unwrapKey(conf, masterKeyName, keyBytes);
108       } catch (KeyException e) {
109         // If the current master key fails to unwrap, try the alternate, if
110         // one is configured
111         if (LOG.isDebugEnabled()) {
112           LOG.debug("Unable to unwrap key with current master key '" + masterKeyName + "'");
113         }
114         String alternateKeyName =
115           conf.get(HConstants.CRYPTO_MASTERKEY_ALTERNATE_NAME_CONF_KEY);
116         if (alternateKeyName != null) {
117           try {
118             key = EncryptionUtil.unwrapKey(conf, alternateKeyName, keyBytes);
119           } catch (KeyException ex) {
120             throw new IOException(ex);
121           }
122         } else {
123           throw new IOException(e);
124         }
125       }
126       // Use the algorithm the key wants
127       Cipher cipher = Encryption.getCipher(conf, key.getAlgorithm());
128       if (cipher == null) {
129         throw new IOException("Cipher '" + key.getAlgorithm() + "' is not available");
130       }
131       cryptoContext.setCipher(cipher);
132       cryptoContext.setKey(key);
133       builder.withEncryptionContext(cryptoContext);
134     }
135 
136     HFileContext context = builder.build();
137 
138     if (LOG.isTraceEnabled()) {
139       LOG.trace("Reader" + (path != null ? " for " + path : "" ) +
140         " initialized with cacheConf: " + cacheConf +
141         " comparator: " + comparator.getClass().getSimpleName() +
142         " fileContext: " + context);
143     }
144 
145     return context;
146   }
147 
148   /**
149    * Create a Scanner on this file. No seeks or reads are done on creation. Call
150    * {@link HFileScanner#seekTo(byte[])} to position an start the read. There is
151    * nothing to clean up in a Scanner. Letting go of your references to the
152    * scanner is sufficient.
153    * @param cacheBlocks
154    *          True if we should cache blocks read in by this scanner.
155    * @param pread
156    *          Use positional read rather than seek+read if true (pread is better
157    *          for random reads, seek+read is better scanning).
158    * @param isCompaction
159    *          is scanner being used for a compaction?
160    * @return Scanner on this file.
161    */
162   @Override
163   public HFileScanner getScanner(boolean cacheBlocks, final boolean pread,
164       final boolean isCompaction) {
165     if (dataBlockEncoder.useEncodedScanner()) {
166       return new EncodedScannerV3(this, cacheBlocks, pread, isCompaction, this.hfileContext);
167     }
168     return new ScannerV3(this, cacheBlocks, pread, isCompaction);
169   }
170 
171   /**
172    * Implementation of {@link HFileScanner} interface.
173    */
174   protected static class ScannerV3 extends ScannerV2 {
175 
176     private HFileReaderV3 reader;
177     private int currTagsLen;
178 
179     public ScannerV3(HFileReaderV3 r, boolean cacheBlocks, final boolean pread,
180         final boolean isCompaction) {
181       super(r, cacheBlocks, pread, isCompaction);
182       this.reader = r;
183     }
184 
185     @Override
186     protected int getCellBufSize() {
187       int kvBufSize = super.getCellBufSize();
188       if (reader.hfileContext.isIncludesTags()) {
189         kvBufSize += Bytes.SIZEOF_SHORT + currTagsLen;
190       }
191       return kvBufSize;
192     }
193 
194     @Override
195     public Cell getKeyValue() {
196       if (!isSeeked())
197         return null;
198       if (currTagsLen > 0) {
199         KeyValue ret = new KeyValue(blockBuffer.array(), blockBuffer.arrayOffset()
200             + blockBuffer.position(), getCellBufSize());
201         if (this.reader.shouldIncludeMemstoreTS()) {
202           ret.setSequenceId(currMemstoreTS);
203         }
204         return ret;
205       } else {
206         return formNoTagsKeyValue();
207       }
208     }
209 
210     protected void setNonSeekedState() {
211       super.setNonSeekedState();
212       currTagsLen = 0;
213     }
214 
215     @Override
216     protected int getNextCellStartPosition() {
217       int nextKvPos = super.getNextCellStartPosition();
218       if (reader.hfileContext.isIncludesTags()) {
219         nextKvPos += Bytes.SIZEOF_SHORT + currTagsLen;
220       }
221       return nextKvPos;
222     }
223 
224     private final void checkTagsLen() {
225       if (checkLen(this.currTagsLen)) {
226         throw new IllegalStateException("Invalid currTagsLen " + this.currTagsLen +
227           ". Block offset: " + block.getOffset() + ", block length: " + this.blockBuffer.limit() +
228           ", position: " + this.blockBuffer.position() + " (without header).");
229       }
230     }
231 
232     protected final void readKeyValueLen() {
233       // TODO: METHOD (mostly) DUPLICATED IN V2!!!! FIXED in master branch by collapsing v3 and v2.
234       // This is a hot method. We go out of our way to make this method short so it can be
235       // inlined and is not too big to compile. We also manage position in ByteBuffer ourselves
236       // because it is faster than going via range-checked ByteBuffer methods or going through a
237       // byte buffer array a byte at a time.
238       int p = blockBuffer.position() + blockBuffer.arrayOffset();
239       // Get a long at a time rather than read two individual ints. In micro-benchmarking, even
240       // with the extra bit-fiddling, this is order-of-magnitude faster than getting two ints.
241       long ll = Bytes.toLong(blockBuffer.array(), p);
242       // Read top half as an int of key length and bottom int as value length
243       this.currKeyLen = (int)(ll >> Integer.SIZE);
244       this.currValueLen = (int)(Bytes.MASK_FOR_LOWER_INT_IN_LONG ^ ll);
245       checkKeyValueLen();
246       // Move position past the key and value lengths and then beyond the key and value
247       p += (Bytes.SIZEOF_LONG + currKeyLen + currValueLen);
248       if (reader.hfileContext.isIncludesTags()) {
249         // Tags length is a short.
250         this.currTagsLen = Bytes.toShort(blockBuffer.array(), p);
251         checkTagsLen();
252         p += (Bytes.SIZEOF_SHORT + currTagsLen);
253       }
254       readMvccVersion(p);
255     }
256 
257     /**
258      * Within a loaded block, seek looking for the last key that is smaller than
259      * (or equal to?) the key we are interested in.
260      * A note on the seekBefore: if you have seekBefore = true, AND the first
261      * key in the block = key, then you'll get thrown exceptions. The caller has
262      * to check for that case and load the previous block as appropriate.
263      * @param key
264      *          the key to find
265      * @param seekBefore
266      *          find the key before the given key in case of exact match.
267      * @return 0 in case of an exact key match, 1 in case of an inexact match,
268      *         -2 in case of an inexact match and furthermore, the input key
269      *         less than the first key of current block(e.g. using a faked index
270      *         key)
271      */
272     @Override
273     protected int blockSeek(Cell key, boolean seekBefore) {
274       int klen, vlen, tlen = 0;
275       long memstoreTS = 0;
276       int memstoreTSLen = 0;
277       int lastKeyValueSize = -1;
278       KeyValue.KeyOnlyKeyValue keyOnlyKv = new KeyValue.KeyOnlyKeyValue();
279       do {
280         blockBuffer.mark();
281         klen = blockBuffer.getInt();
282         vlen = blockBuffer.getInt();
283         if (klen < 0 || vlen < 0 || klen > blockBuffer.limit()
284             || vlen > blockBuffer.limit()) {
285           throw new IllegalStateException("Invalid klen " + klen + " or vlen "
286               + vlen + ". Block offset: "
287               + block.getOffset() + ", block length: " + blockBuffer.limit() + ", position: "
288               + blockBuffer.position() + " (without header).");
289         }
290         ByteBufferUtils.skip(blockBuffer, klen + vlen);
291         if (reader.hfileContext.isIncludesTags()) {
292           // Read short as unsigned, high byte first
293           tlen = ((blockBuffer.get() & 0xff) << 8) ^ (blockBuffer.get() & 0xff);
294           if (tlen < 0 || tlen > blockBuffer.limit()) {
295             throw new IllegalStateException("Invalid tlen " + tlen + ". Block offset: "
296                 + block.getOffset() + ", block length: " + blockBuffer.limit() + ", position: "
297                 + blockBuffer.position() + " (without header).");
298           }
299           ByteBufferUtils.skip(blockBuffer, tlen);
300         }
301         if (this.reader.shouldIncludeMemstoreTS()) {
302           if (this.reader.decodeMemstoreTS) {
303             memstoreTS = Bytes.readAsVLong(blockBuffer.array(), blockBuffer.arrayOffset()
304                 + blockBuffer.position());
305             memstoreTSLen = WritableUtils.getVIntSize(memstoreTS);
306           } else {
307             memstoreTS = 0;
308             memstoreTSLen = 1;
309           }
310         }
311         blockBuffer.reset();
312         int keyOffset =
313           blockBuffer.arrayOffset() + blockBuffer.position() + (Bytes.SIZEOF_INT * 2);
314         keyOnlyKv.setKey(blockBuffer.array(), keyOffset, klen);
315         int comp = reader.getComparator().compareOnlyKeyPortion(key, keyOnlyKv);
316 
317         if (comp == 0) {
318           if (seekBefore) {
319             if (lastKeyValueSize < 0) {
320               throw new IllegalStateException("blockSeek with seekBefore "
321                   + "at the first key of the block: key="
322                   + CellUtil.getCellKeyAsString(key)
323                   + ", blockOffset=" + block.getOffset() + ", onDiskSize="
324                   + block.getOnDiskSizeWithHeader());
325             }
326             blockBuffer.position(blockBuffer.position() - lastKeyValueSize);
327             readKeyValueLen();
328             return 1; // non exact match.
329           }
330           currKeyLen = klen;
331           currValueLen = vlen;
332           currTagsLen = tlen;
333           if (this.reader.shouldIncludeMemstoreTS()) {
334             currMemstoreTS = memstoreTS;
335             currMemstoreTSLen = memstoreTSLen;
336           }
337           return 0; // indicate exact match
338         } else if (comp < 0) {
339           if (lastKeyValueSize > 0)
340             blockBuffer.position(blockBuffer.position() - lastKeyValueSize);
341           readKeyValueLen();
342           if (lastKeyValueSize == -1 && blockBuffer.position() == 0) {
343             return HConstants.INDEX_KEY_MAGIC;
344           }
345           return 1;
346         }
347 
348         // The size of this key/value tuple, including key/value length fields.
349         lastKeyValueSize = klen + vlen + memstoreTSLen + KEY_VALUE_LEN_SIZE;
350         // include tag length also if tags included with KV
351         if (reader.hfileContext.isIncludesTags()) {
352           lastKeyValueSize += tlen + Bytes.SIZEOF_SHORT;
353         }
354         blockBuffer.position(blockBuffer.position() + lastKeyValueSize);
355       } while (blockBuffer.remaining() > 0);
356 
357       // Seek to the last key we successfully read. This will happen if this is
358       // the last key/value pair in the file, in which case the following call
359       // to next() has to return false.
360       blockBuffer.position(blockBuffer.position() - lastKeyValueSize);
361       readKeyValueLen();
362       return 1; // didn't exactly find it.
363     }
364   }
365 
366   /**
367    * ScannerV3 that operates on encoded data blocks.
368    */
369   protected static class EncodedScannerV3 extends EncodedScannerV2 {
370     public EncodedScannerV3(HFileReaderV3 reader, boolean cacheBlocks, boolean pread,
371         boolean isCompaction, HFileContext context) {
372       super(reader, cacheBlocks, pread, isCompaction, context);
373     }
374   }
375 
376   @Override
377   public int getMajorVersion() {
378     return 3;
379   }
380 }