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.types;
019
020import java.util.Iterator;
021
022import org.apache.hadoop.hbase.util.Order;
023import org.apache.hadoop.hbase.util.PositionedByteRange;
024import org.apache.yetus.audience.InterfaceAudience;
025
026/**
027 * <p>
028 * {@code Struct} is a simple {@link DataType} for implementing "compound
029 * rowkey" and "compound qualifier" schema design strategies.
030 * </p>
031 * <h3>Encoding</h3>
032 * <p>
033 * {@code Struct} member values are encoded onto the target byte[] in the order
034 * in which they are declared. A {@code Struct} may be used as a member of
035 * another {@code Struct}. {@code Struct}s are not {@code nullable} but their
036 * component fields may be.
037 * </p>
038 * <h3>Trailing Nulls</h3>
039 * <p>
040 * {@code Struct} treats the right-most nullable field members as special.
041 * Rather than writing null values to the output buffer, {@code Struct} omits
042 * those records all together. When reading back a value, it will look for the
043 * scenario where the end of the buffer has been reached but there are still
044 * nullable fields remaining in the {@code Struct} definition. When this
045 * happens, it will produce null entries for the remaining values. For example:
046 * </p>
047 * <pre>
048 * StructBuilder builder = new StructBuilder()
049 *     .add(OrderedNumeric.ASCENDING) // nullable
050 *     .add(OrderedString.ASCENDING)  // nullable
051 * Struct shorter = builder.toStruct();
052 * Struct longer = builder.add(OrderedNumeric.ASCENDING) // nullable
053 *     .toStruct();
054 *
055 * PositionedByteRange buf1 = new SimplePositionedByteRange(7);
056 * PositionedByteRange buf2 = new SimplePositionedByteRange(7);
057 * Object[] val = new Object[] { BigDecimal.ONE, "foo" };
058 * shorter.encode(buf1, val); // write short value with short Struct
059 * buf1.setPosition(0); // reset position marker, prepare for read
060 * longer.decode(buf1); // =&gt; { BigDecimal.ONE, "foo", null } ; long Struct reads implied null
061 * longer.encode(buf2, val); // write short value with long struct
062 * Bytes.equals(buf1.getBytes(), buf2.getBytes()); // =&gt; true; long Struct skips writing null
063 * </pre>
064 * <h3>Sort Order</h3>
065 * <p>
066 * {@code Struct} instances sort according to the composite order of their
067 * fields, that is, left-to-right and depth-first. This can also be thought of
068 * as lexicographic comparison of concatenated members.
069 * </p>
070 * <p>
071 * {@link StructIterator} is provided as a convenience for consuming the
072 * sequence of values. Users may find it more appropriate to provide their own
073 * custom {@link DataType} for encoding application objects rather than using
074 * this {@code Object[]} implementation. Examples are provided in test.
075 * </p>
076 * @see StructIterator
077 * @see DataType#isNullable()
078 */
079@InterfaceAudience.Public
080public class Struct implements DataType<Object[]> {
081
082  @SuppressWarnings("rawtypes")
083  protected final DataType[] fields;
084  protected final boolean isOrderPreserving;
085  protected final boolean isSkippable;
086
087  /**
088   * Create a new {@code Struct} instance defined as the sequence of
089   * {@code HDataType}s in {@code memberTypes}.
090   * <p>
091   * A {@code Struct} is {@code orderPreserving} when all of its fields
092   * are {@code orderPreserving}. A {@code Struct} is {@code skippable} when
093   * all of its fields are {@code skippable}.
094   * </p>
095   */
096  @SuppressWarnings("rawtypes")
097  public Struct(DataType[] memberTypes) {
098    this.fields = memberTypes;
099    // a Struct is not orderPreserving when any of its fields are not.
100    boolean preservesOrder = true;
101    // a Struct is not skippable when any of its fields are not.
102    boolean skippable = true;
103    for (int i = 0; i < this.fields.length; i++) {
104      DataType dt = this.fields[i];
105      if (!dt.isOrderPreserving()) {
106        preservesOrder = false;
107      }
108      if (i < this.fields.length - 2 && !dt.isSkippable()) {
109        throw new IllegalArgumentException("Field in position " + i
110          + " is not skippable. Non-right-most struct fields must be skippable.");
111      }
112      if (!dt.isSkippable()) {
113        skippable = false;
114      }
115    }
116    this.isOrderPreserving = preservesOrder;
117    this.isSkippable = skippable;
118  }
119
120  @Override
121  public boolean isOrderPreserving() {
122    return isOrderPreserving;
123  }
124
125  @Override
126  public Order getOrder() {
127    return null;
128  }
129
130  @Override
131  public boolean isNullable() {
132    return false;
133  }
134
135  @Override
136  public boolean isSkippable() {
137    return isSkippable;
138  }
139
140  @SuppressWarnings("unchecked")
141  @Override
142  public int encodedLength(Object[] val) {
143    assert fields.length >= val.length;
144    int sum = 0;
145    for (int i = 0; i < val.length; i++) {
146      sum += fields[i].encodedLength(val[i]);
147    }
148    return sum;
149  }
150
151  @Override
152  public Class<Object[]> encodedClass() {
153    return Object[].class;
154  }
155
156  /**
157   * Retrieve an {@link Iterator} over the values encoded in {@code src}.
158   * {@code src}'s position is consumed by consuming this iterator.
159   */
160  public StructIterator iterator(PositionedByteRange src) {
161    return new StructIterator(src, fields);
162  }
163
164  @Override
165  public int skip(PositionedByteRange src) {
166    StructIterator it = iterator(src);
167    int skipped = 0;
168    while (it.hasNext()) {
169      skipped += it.skip();
170    }
171    return skipped;
172  }
173
174  @Override
175  public Object[] decode(PositionedByteRange src) {
176    int i = 0;
177    Object[] ret = new Object[fields.length];
178    Iterator<Object> it = iterator(src);
179    while (it.hasNext()) {
180      ret[i++] = it.next();
181    }
182    return ret;
183  }
184
185  /**
186   * Read the field at {@code index}. {@code src}'s position is not affected.
187   */
188  public Object decode(PositionedByteRange src, int index) {
189    assert index >= 0;
190    StructIterator it = iterator(src.shallowCopy());
191    for (; index > 0; index--) {
192      it.skip();
193    }
194    return it.next();
195  }
196
197  @SuppressWarnings("unchecked")
198  @Override
199  public int encode(PositionedByteRange dst, Object[] val) {
200    if (val.length == 0) {
201      return 0;
202    }
203    assert fields.length >= val.length;
204    int end, written = 0;
205    // find the last occurrence of a non-null or null and non-nullable value
206    for (end = val.length - 1; end > -1; end--) {
207      if (null != val[end] || (null == val[end] && !fields[end].isNullable())) {
208        break;
209      }
210    }
211    for (int i = 0; i <= end; i++) {
212      written += fields[i].encode(dst, val[i]);
213    }
214    return written;
215  }
216}