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;
019
020import java.io.DataInput;
021import java.io.IOException;
022import java.util.Arrays;
023import org.apache.hadoop.fs.FSDataOutputStream;
024import org.apache.hadoop.fs.FileSystem;
025import org.apache.hadoop.fs.Path;
026import org.apache.hadoop.hbase.KeyValueUtil;
027import org.apache.hadoop.hbase.util.Bytes;
028import org.apache.yetus.audience.InterfaceAudience;
029
030import org.apache.hbase.thirdparty.com.google.protobuf.UnsafeByteOperations;
031
032import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
033import org.apache.hadoop.hbase.shaded.protobuf.generated.FSProtos;
034
035/**
036 * A reference to the top or bottom half of a store file where 'bottom' is the first half of the
037 * file containing the keys that sort lowest and 'top' is the second half of the file with keys that
038 * sort greater than those of the bottom half. The file referenced lives under a different region.
039 * References are made at region split time.
040 * <p>
041 * References work with a special half store file type. References know how to write out the
042 * reference format in the file system and are what is juggled when references are mixed in with
043 * direct store files. The half store file type is used reading the referred to file.
044 * <p>
045 * References to store files located over in some other region look like this in the file system
046 * <code>1278437856009925445.3323223323</code>: i.e. an id followed by hash of the referenced
047 * region. Note, a region is itself not splittable if it has instances of store file references.
048 * References are cleaned up by compactions.
049 */
050@InterfaceAudience.Private
051public class Reference {
052  private byte[] splitkey;
053  private Range region;
054
055  /**
056   * For split HStoreFiles, it specifies if the file covers the lower half or the upper half of the
057   * key range
058   */
059  static enum Range {
060    /** HStoreFile contains upper half of key range */
061    top,
062    /** HStoreFile contains lower half of key range */
063    bottom
064  }
065
066  /** Returns A {@link Reference} that points at top half of a an hfile */
067  public static Reference createTopReference(final byte[] splitRow) {
068    return new Reference(splitRow, Range.top);
069  }
070
071  /** Returns A {@link Reference} that points at the bottom half of a an hfile */
072  public static Reference createBottomReference(final byte[] splitRow) {
073    return new Reference(splitRow, Range.bottom);
074  }
075
076  /**
077   * Constructor
078   * @param splitRow This is row we are splitting around.
079   */
080  Reference(final byte[] splitRow, final Range fr) {
081    this.splitkey = splitRow == null ? null : KeyValueUtil.createFirstOnRow(splitRow).getKey();
082    this.region = fr;
083  }
084
085  /**
086   * Used by serializations.
087   * @deprecated need by pb serialization
088   */
089  @Deprecated
090  // Make this private when it comes time to let go of this constructor.
091  // Needed by pb serialization.
092  public Reference() {
093    this(null, Range.bottom);
094  }
095
096  /**
097   *   */
098  public Range getFileRegion() {
099    return this.region;
100  }
101
102  /**
103   *   */
104  public byte[] getSplitKey() {
105    return splitkey;
106  }
107
108  /**
109   * @see java.lang.Object#toString()
110   */
111  @Override
112  public String toString() {
113    return "" + this.region;
114  }
115
116  public static boolean isTopFileRegion(final Range r) {
117    return r.equals(Range.top);
118  }
119
120  /**
121   * @deprecated Writables are going away. Use the pb serialization methods instead. Remove in a
122   *             release after 0.96 goes out. This is here only to migrate old Reference files
123   *             written with Writables before 0.96.
124   */
125  @Deprecated
126  public void readFields(DataInput in) throws IOException {
127    boolean tmp = in.readBoolean();
128    // If true, set region to top.
129    this.region = tmp ? Range.top : Range.bottom;
130    this.splitkey = Bytes.readByteArray(in);
131  }
132
133  public Path write(final FileSystem fs, final Path p) throws IOException {
134    FSDataOutputStream out = fs.create(p, false);
135    try {
136      out.write(toByteArray());
137    } finally {
138      out.close();
139    }
140    return p;
141  }
142
143  public FSProtos.Reference convert() {
144    FSProtos.Reference.Builder builder = FSProtos.Reference.newBuilder();
145    builder.setRange(isTopFileRegion(getFileRegion())
146      ? FSProtos.Reference.Range.TOP
147      : FSProtos.Reference.Range.BOTTOM);
148    builder.setSplitkey(UnsafeByteOperations.unsafeWrap(getSplitKey()));
149    return builder.build();
150  }
151
152  public static Reference convert(final FSProtos.Reference r) {
153    Reference result = new Reference();
154    result.splitkey = r.getSplitkey().toByteArray();
155    result.region = r.getRange() == FSProtos.Reference.Range.TOP ? Range.top : Range.bottom;
156    return result;
157  }
158
159  /**
160   * Use this when writing to a stream and you want to use the pb mergeDelimitedFrom (w/o the
161   * delimiter, pb reads to EOF which may not be what you want).
162   * @return This instance serialized as a delimited protobuf w/ a magic pb prefix.
163   */
164  public byte[] toByteArray() throws IOException {
165    return ProtobufUtil.prependPBMagic(convert().toByteArray());
166  }
167
168  @Override
169  public int hashCode() {
170    return Arrays.hashCode(splitkey) + region.hashCode();
171  }
172
173  @Override
174  public boolean equals(Object o) {
175    if (this == o) return true;
176    if (o == null) return false;
177    if (!(o instanceof Reference)) return false;
178
179    Reference r = (Reference) o;
180    if (splitkey != null && r.splitkey == null) return false;
181    if (splitkey == null && r.splitkey != null) return false;
182    if (splitkey != null && !Arrays.equals(splitkey, r.splitkey)) return false;
183
184    return region.equals(r.region);
185  }
186}