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.filter;
020
021import java.io.IOException;
022import java.nio.ByteBuffer;
023import java.util.ArrayList;
024import java.util.Collections;
025import java.util.Iterator;
026import java.util.Objects;
027import java.util.Optional;
028
029import org.apache.hadoop.hbase.ByteBufferExtendedCell;
030import org.apache.hadoop.hbase.Cell;
031import org.apache.hadoop.hbase.HConstants;
032import org.apache.hadoop.hbase.KeyValue;
033import org.apache.hadoop.hbase.KeyValueUtil;
034import org.apache.hadoop.hbase.Tag;
035import org.apache.hadoop.hbase.exceptions.DeserializationException;
036import org.apache.hadoop.hbase.util.Bytes;
037import org.apache.hadoop.hbase.util.ClassSize;
038import org.apache.yetus.audience.InterfaceAudience;
039
040import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
041import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException;
042import org.apache.hadoop.hbase.shaded.protobuf.generated.FilterProtos;
043
044/**
045 * A filter that will only return the key component of each KV (the value will
046 * be rewritten as empty).
047 * <p>
048 * This filter can be used to grab all of the keys without having to also grab
049 * the values.
050 */
051@InterfaceAudience.Public
052public class KeyOnlyFilter extends FilterBase {
053
054  boolean lenAsVal;
055  public KeyOnlyFilter() { this(false); }
056  public KeyOnlyFilter(boolean lenAsVal) { this.lenAsVal = lenAsVal; }
057
058  @Override
059  public boolean filterRowKey(Cell cell) throws IOException {
060    // Impl in FilterBase might do unnecessary copy for Off heap backed Cells.
061    return false;
062  }
063
064  @Override
065  public Cell transformCell(Cell cell) {
066    return createKeyOnlyCell(cell);
067  }
068
069  private Cell createKeyOnlyCell(Cell c) {
070    if (c instanceof ByteBufferExtendedCell) {
071      return new KeyOnlyByteBufferExtendedCell((ByteBufferExtendedCell) c, lenAsVal);
072    } else {
073      return new KeyOnlyCell(c, lenAsVal);
074    }
075  }
076
077  @Deprecated
078  @Override
079  public ReturnCode filterKeyValue(final Cell ignored) throws IOException {
080    return filterCell(ignored);
081  }
082
083  @Override
084  public ReturnCode filterCell(final Cell ignored) throws IOException {
085    return ReturnCode.INCLUDE;
086  }
087
088  public static Filter createFilterFromArguments(ArrayList<byte []> filterArguments) {
089    Preconditions.checkArgument((filterArguments.isEmpty() || filterArguments.size() == 1),
090                                "Expected: 0 or 1 but got: %s", filterArguments.size());
091    KeyOnlyFilter filter = new KeyOnlyFilter();
092    if (filterArguments.size() == 1) {
093      filter.lenAsVal = ParseFilter.convertByteArrayToBoolean(filterArguments.get(0));
094    }
095    return filter;
096  }
097
098  /**
099   * @return The filter serialized using pb
100   */
101  @Override
102  public byte [] toByteArray() {
103    FilterProtos.KeyOnlyFilter.Builder builder =
104      FilterProtos.KeyOnlyFilter.newBuilder();
105    builder.setLenAsVal(this.lenAsVal);
106    return builder.build().toByteArray();
107  }
108
109  /**
110   * @param pbBytes A pb serialized {@link KeyOnlyFilter} instance
111   * @return An instance of {@link KeyOnlyFilter} made from <code>bytes</code>
112   * @throws DeserializationException
113   * @see #toByteArray
114   */
115  public static KeyOnlyFilter parseFrom(final byte [] pbBytes)
116  throws DeserializationException {
117    FilterProtos.KeyOnlyFilter proto;
118    try {
119      proto = FilterProtos.KeyOnlyFilter.parseFrom(pbBytes);
120    } catch (InvalidProtocolBufferException e) {
121      throw new DeserializationException(e);
122    }
123    return new KeyOnlyFilter(proto.getLenAsVal());
124  }
125
126  /**
127   * @param o the other filter to compare with
128   * @return true if and only if the fields of the filter that are serialized
129   * are equal to the corresponding fields in other.  Used for testing.
130   */
131  @Override
132  boolean areSerializedFieldsEqual(Filter o) {
133    if (o == this) return true;
134    if (!(o instanceof KeyOnlyFilter)) return false;
135
136    KeyOnlyFilter other = (KeyOnlyFilter)o;
137    return this.lenAsVal == other.lenAsVal;
138  }
139
140  @Override
141  public boolean equals(Object obj) {
142    return obj instanceof Filter && areSerializedFieldsEqual((Filter) obj);
143  }
144
145  @Override
146  public int hashCode() {
147    return Objects.hash(this.lenAsVal);
148  }
149
150  static class KeyOnlyCell implements Cell {
151    private Cell cell;
152    private int keyLen;
153    private boolean lenAsVal;
154
155    public KeyOnlyCell(Cell c, boolean lenAsVal) {
156      this.cell = c;
157      this.lenAsVal = lenAsVal;
158      this.keyLen = KeyValueUtil.keyLength(c);
159    }
160
161    @Override
162    public byte[] getRowArray() {
163      return cell.getRowArray();
164    }
165
166    @Override
167    public int getRowOffset() {
168      return cell.getRowOffset();
169    }
170
171    @Override
172    public short getRowLength() {
173      return cell.getRowLength();
174    }
175
176    @Override
177    public byte[] getFamilyArray() {
178      return cell.getFamilyArray();
179    }
180
181    @Override
182    public int getFamilyOffset() {
183      return cell.getFamilyOffset();
184    }
185
186    @Override
187    public byte getFamilyLength() {
188      return cell.getFamilyLength();
189    }
190
191    @Override
192    public byte[] getQualifierArray() {
193      return cell.getQualifierArray();
194    }
195
196    @Override
197    public int getQualifierOffset() {
198      return cell.getQualifierOffset();
199    }
200
201    @Override
202    public int getQualifierLength() {
203      return cell.getQualifierLength();
204    }
205
206    @Override
207    public long getTimestamp() {
208      return cell.getTimestamp();
209    }
210
211    @Override
212    public byte getTypeByte() {
213      return cell.getTypeByte();
214    }
215
216    @Override
217    public Type getType() {
218      return cell.getType();
219    }
220
221
222    @Override
223    public long getSequenceId() {
224      return 0;
225    }
226
227    @Override
228    public byte[] getValueArray() {
229      if (lenAsVal) {
230        return Bytes.toBytes(cell.getValueLength());
231      } else {
232        return HConstants.EMPTY_BYTE_ARRAY;
233      }
234    }
235
236    @Override
237    public int getValueOffset() {
238      return 0;
239    }
240
241    @Override
242    public int getValueLength() {
243      if (lenAsVal) {
244        return Bytes.SIZEOF_INT;
245      } else {
246        return 0;
247      }
248    }
249
250    @Override
251    public int getSerializedSize() {
252      return KeyValue.KEYVALUE_INFRASTRUCTURE_SIZE + keyLen + getValueLength();
253    }
254
255    @Override
256    public byte[] getTagsArray() {
257      return HConstants.EMPTY_BYTE_ARRAY;
258    }
259
260    @Override
261    public int getTagsOffset() {
262      return 0;
263    }
264
265    @Override
266    public int getTagsLength() {
267      return 0;
268    }
269
270    @Override
271    public long heapSize() {
272      return cell.heapSize();
273    }
274  }
275
276  static class KeyOnlyByteBufferExtendedCell extends ByteBufferExtendedCell {
277    public static final int FIXED_OVERHEAD = ClassSize.OBJECT + ClassSize.REFERENCE
278        + Bytes.SIZEOF_BOOLEAN;
279    private ByteBufferExtendedCell cell;
280    private boolean lenAsVal;
281
282    public KeyOnlyByteBufferExtendedCell(ByteBufferExtendedCell c, boolean lenAsVal) {
283      this.cell = c;
284      this.lenAsVal = lenAsVal;
285    }
286
287    @Override
288    public byte[] getRowArray() {
289      return cell.getRowArray();
290    }
291
292    @Override
293    public int getRowOffset() {
294      return cell.getRowOffset();
295    }
296
297    @Override
298    public short getRowLength() {
299      return cell.getRowLength();
300    }
301
302    @Override
303    public byte[] getFamilyArray() {
304      return cell.getFamilyArray();
305    }
306
307    @Override
308    public int getFamilyOffset() {
309      return cell.getFamilyOffset();
310    }
311
312    @Override
313    public byte getFamilyLength() {
314      return cell.getFamilyLength();
315    }
316
317    @Override
318    public byte[] getQualifierArray() {
319      return cell.getQualifierArray();
320    }
321
322    @Override
323    public int getQualifierOffset() {
324      return cell.getQualifierOffset();
325    }
326
327    @Override
328    public int getQualifierLength() {
329      return cell.getQualifierLength();
330    }
331
332    @Override
333    public long getTimestamp() {
334      return cell.getTimestamp();
335    }
336
337    @Override
338    public byte getTypeByte() {
339      return cell.getTypeByte();
340    }
341
342    @Override
343    public void setSequenceId(long seqId) throws IOException {
344      cell.setSequenceId(seqId);
345    }
346
347    @Override
348    public void setTimestamp(long ts) throws IOException {
349      cell.setTimestamp(ts);
350    }
351
352    @Override
353    public void setTimestamp(byte[] ts) throws IOException {
354      cell.setTimestamp(ts);
355    }
356
357    @Override
358    public long getSequenceId() {
359      return 0;
360    }
361
362    @Override
363    public Type getType() {
364      return cell.getType();
365    }
366
367    @Override
368    public byte[] getValueArray() {
369      if (lenAsVal) {
370        return Bytes.toBytes(cell.getValueLength());
371      } else {
372        return HConstants.EMPTY_BYTE_ARRAY;
373      }
374    }
375
376    @Override
377    public int getValueOffset() {
378      return 0;
379    }
380
381    @Override
382    public int getValueLength() {
383      if (lenAsVal) {
384        return Bytes.SIZEOF_INT;
385      } else {
386        return 0;
387      }
388    }
389
390    @Override
391    public byte[] getTagsArray() {
392      return HConstants.EMPTY_BYTE_ARRAY;
393    }
394
395    @Override
396    public int getTagsOffset() {
397      return 0;
398    }
399
400    @Override
401    public int getTagsLength() {
402      return 0;
403    }
404
405    @Override
406    public ByteBuffer getRowByteBuffer() {
407      return cell.getRowByteBuffer();
408    }
409
410    @Override
411    public int getRowPosition() {
412      return cell.getRowPosition();
413    }
414
415    @Override
416    public ByteBuffer getFamilyByteBuffer() {
417      return cell.getFamilyByteBuffer();
418    }
419
420    @Override
421    public int getFamilyPosition() {
422      return cell.getFamilyPosition();
423    }
424
425    @Override
426    public ByteBuffer getQualifierByteBuffer() {
427      return cell.getQualifierByteBuffer();
428    }
429
430    @Override
431    public int getQualifierPosition() {
432      return cell.getQualifierPosition();
433    }
434
435    @Override
436    public ByteBuffer getValueByteBuffer() {
437      if (lenAsVal) {
438        return ByteBuffer.wrap(Bytes.toBytes(cell.getValueLength()));
439      } else {
440        return HConstants.EMPTY_BYTE_BUFFER;
441      }
442    }
443
444    @Override
445    public int getValuePosition() {
446      return 0;
447    }
448
449    @Override
450    public ByteBuffer getTagsByteBuffer() {
451      return HConstants.EMPTY_BYTE_BUFFER;
452    }
453
454    @Override
455    public int getTagsPosition() {
456      return 0;
457    }
458
459    @Override
460    public Iterator<Tag> getTags() {
461      return Collections.emptyIterator();
462    }
463
464    @Override
465    public Optional<Tag> getTag(byte type) {
466      return Optional.empty();
467    }
468
469    @Override
470    public long heapSize() {
471      return ClassSize.align(FIXED_OVERHEAD + cell.heapSize());
472    }
473  }
474
475}