Skip to content

Commit

Permalink
HBASE-28556 Reduce memory copying in Rest server when serializing Cel…
Browse files Browse the repository at this point in the history
…lModel to Protobuf (#5870)

Signed-off-by: Duo Zhang <zhangduo@apache.org>
  • Loading branch information
stoty committed May 7, 2024
1 parent 8b9cafc commit fcfc112
Show file tree
Hide file tree
Showing 9 changed files with 215 additions and 73 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -22,14 +22,10 @@
import java.util.Base64;
import java.util.Base64.Decoder;
import java.util.List;
import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.CellUtil;
import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.filter.Filter;
import org.apache.hadoop.hbase.filter.ParseFilter;
import org.apache.hadoop.hbase.rest.model.CellModel;
import org.apache.hadoop.hbase.rest.model.CellSetModel;
import org.apache.hadoop.hbase.rest.model.RowModel;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
Expand Down Expand Up @@ -125,12 +121,7 @@ public Response get(final @Context UriInfo uriInfo,
if (r.isEmpty()) {
continue;
}
RowModel rowModel = new RowModel(r.getRow());
for (Cell c : r.listCells()) {
rowModel.addCell(new CellModel(CellUtil.cloneFamily(c), CellUtil.cloneQualifier(c),
c.getTimestamp(), CellUtil.cloneValue(c)));
}
model.addRow(rowModel);
model.addRow(RestUtil.createRowModelFromResult(r));
}
if (model.getRows().isEmpty()) {
// If no rows found.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,14 +19,9 @@

import java.io.IOException;
import java.io.OutputStream;
import java.util.List;
import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.CellUtil;
import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.client.ResultScanner;
import org.apache.hadoop.hbase.rest.model.CellModel;
import org.apache.hadoop.hbase.rest.model.CellSetModel;
import org.apache.hadoop.hbase.rest.model.RowModel;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
Expand Down Expand Up @@ -91,15 +86,11 @@ private void writeToStream(CellSetModel model, String contentType, OutputStream

private CellSetModel createModelFromResults(Result[] results) {
CellSetModel cellSetModel = new CellSetModel();
for (Result rs : results) {
byte[] rowKey = rs.getRow();
RowModel rModel = new RowModel(rowKey);
List<Cell> kvs = rs.listCells();
for (Cell kv : kvs) {
rModel.addCell(new CellModel(CellUtil.cloneFamily(kv), CellUtil.cloneQualifier(kv),
kv.getTimestamp(), CellUtil.cloneValue(kv)));
for (int i = 0; i < results.length; i++) {
if (results[i].isEmpty()) {
continue;
}
cellSetModel.addRow(rModel);
cellSetModel.addRow(RestUtil.createRowModelFromResult(results[i]));
}
return cellSetModel;
}
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,48 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hbase.rest;

import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.rest.model.CellModel;
import org.apache.hadoop.hbase.rest.model.RowModel;
import org.apache.yetus.audience.InterfaceAudience;

@InterfaceAudience.Private
public final class RestUtil {

private RestUtil() {
// Do not instantiate
}

/**
* Speed-optimized method to convert an HBase result to a RowModel. Avoids iterators and uses the
* non-cloning constructors to minimize overhead, especially when using protobuf marshalling.
* @param r non-empty Result object
*/
public static RowModel createRowModelFromResult(Result r) {
Cell firstCell = r.rawCells()[0];
RowModel rowModel =
new RowModel(firstCell.getRowArray(), firstCell.getRowOffset(), firstCell.getRowLength());
int cellsLength = r.rawCells().length;
for (int i = 0; i < cellsLength; i++) {
rowModel.addCell(new CellModel(r.rawCells()[i]));
}
return rowModel;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -112,8 +112,7 @@ public Response get(final @Context UriInfo uriInfo) {
rowKey = CellUtil.cloneRow(value);
rowModel = new RowModel(rowKey);
}
rowModel.addCell(new CellModel(CellUtil.cloneFamily(value), CellUtil.cloneQualifier(value),
value.getTimestamp(), CellUtil.cloneValue(value)));
rowModel.addCell(new CellModel(value));
if (++count > rowspec.getMaxValues()) {
break;
}
Expand Down Expand Up @@ -711,8 +710,7 @@ Response append(final CellSetModel model) {
CellSetModel rModel = new CellSetModel();
RowModel rRowModel = new RowModel(result.getRow());
for (Cell cell : result.listCells()) {
rRowModel.addCell(new CellModel(CellUtil.cloneFamily(cell), CellUtil.cloneQualifier(cell),
cell.getTimestamp(), CellUtil.cloneValue(cell)));
rRowModel.addCell(new CellModel(cell));
}
rModel.addRow(rRowModel);
servlet.getMetrics().incrementSucessfulAppendRequests(1);
Expand Down Expand Up @@ -803,8 +801,7 @@ Response increment(final CellSetModel model) {
CellSetModel rModel = new CellSetModel();
RowModel rRowModel = new RowModel(result.getRow());
for (Cell cell : result.listCells()) {
rRowModel.addCell(new CellModel(CellUtil.cloneFamily(cell), CellUtil.cloneQualifier(cell),
cell.getTimestamp(), CellUtil.cloneValue(cell)));
rRowModel.addCell(new CellModel(cell));
}
rModel.addRow(rowModel);
servlet.getMetrics().incrementSucessfulIncrementRequests(1);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -83,7 +83,9 @@ public Response get(final @Context UriInfo uriInfo, @QueryParam("n") int maxRows
}
CellSetModel model = new CellSetModel();
RowModel rowModel = null;
byte[] rowKey = null;
byte[] rowKeyArray = null;
int rowKeyOffset = 0;
int rowKeyLength = 0;
int limit = batch;
if (maxValues > 0) {
limit = maxValues;
Expand Down Expand Up @@ -121,11 +123,13 @@ public Response get(final @Context UriInfo uriInfo, @QueryParam("n") int maxRows
}
break;
}
if (rowKey == null) {
rowKey = CellUtil.cloneRow(value);
rowModel = new RowModel(rowKey);
if (rowKeyArray == null) {
rowKeyArray = value.getRowArray();
rowKeyOffset = value.getRowOffset();
rowKeyLength = value.getRowLength();
rowModel = new RowModel(rowKeyArray, rowKeyOffset, rowKeyLength);
}
if (!Bytes.equals(CellUtil.cloneRow(value), rowKey)) {
if (!CellUtil.matchingRow(value, rowKeyArray, rowKeyOffset, rowKeyLength)) {
// if maxRows was given as a query param, stop if we would exceed the
// specified number of rows
if (maxRows > 0) {
Expand All @@ -135,11 +139,12 @@ public Response get(final @Context UriInfo uriInfo, @QueryParam("n") int maxRows
}
}
model.addRow(rowModel);
rowKey = CellUtil.cloneRow(value);
rowModel = new RowModel(rowKey);
rowKeyArray = value.getRowArray();
rowKeyOffset = value.getRowOffset();
rowKeyLength = value.getRowLength();
rowModel = new RowModel(rowKeyArray, rowKeyOffset, rowKeyLength);
}
rowModel.addCell(new CellModel(CellUtil.cloneFamily(value), CellUtil.cloneQualifier(value),
value.getTimestamp(), CellUtil.cloneValue(value)));
rowModel.addCell(new CellModel(value));
} while (--count > 0);
model.addRow(rowModel);
ResponseBuilder response = Response.ok(model);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,16 +22,12 @@
import java.io.IOException;
import java.util.ArrayList;
import java.util.Iterator;
import java.util.List;
import javax.xml.bind.annotation.XmlAccessType;
import javax.xml.bind.annotation.XmlAccessorType;
import javax.xml.bind.annotation.XmlElement;
import javax.xml.bind.annotation.XmlRootElement;
import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.CellUtil;
import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.client.ResultScanner;
import org.apache.hadoop.hbase.rest.model.CellModel;
import org.apache.hadoop.hbase.rest.model.RowModel;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
Expand Down Expand Up @@ -87,13 +83,7 @@ public RowModel next() {
if ((rs == null) || (count <= 0)) {
return null;
}
byte[] rowKey = rs.getRow();
RowModel rModel = new RowModel(rowKey);
List<Cell> kvs = rs.listCells();
for (Cell kv : kvs) {
rModel.addCell(new CellModel(CellUtil.cloneFamily(kv), CellUtil.cloneQualifier(kv),
kv.getTimestamp(), CellUtil.cloneValue(kv)));
}
RowModel rModel = RestUtil.createRowModelFromResult(rs);
count--;
if (count == 0) {
results.close();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,9 @@
*/
package org.apache.hadoop.hbase.rest.model;

import static org.apache.hadoop.hbase.KeyValue.COLUMN_FAMILY_DELIMITER;

import com.fasterxml.jackson.annotation.JsonIgnore;
import com.fasterxml.jackson.annotation.JsonProperty;
import java.io.IOException;
import java.io.Serializable;
Expand Down Expand Up @@ -56,10 +59,11 @@
* </pre>
*/
@XmlRootElement(name = "Cell")
@XmlAccessorType(XmlAccessType.FIELD)
@XmlAccessorType(XmlAccessType.NONE)
@InterfaceAudience.Private
public class CellModel implements ProtobufMessageHandler, Serializable {
private static final long serialVersionUID = 1L;
public static final int MAGIC_LENGTH = -1;

@JsonProperty("column")
@XmlAttribute
Expand All @@ -69,10 +73,17 @@ public class CellModel implements ProtobufMessageHandler, Serializable {
@XmlAttribute
private long timestamp = HConstants.LATEST_TIMESTAMP;

@JsonProperty("$")
@XmlValue
// If valueLength = -1, this represents the cell's value.
// If valueLength <> 1, this represents an array containing the cell's value as determined by
// offset and length.
private byte[] value;

@JsonIgnore
private int valueOffset;

@JsonIgnore
private int valueLength = MAGIC_LENGTH;

/**
* Default constructor
*/
Expand All @@ -94,11 +105,16 @@ public CellModel(byte[] column, byte[] qualifier, byte[] value) {
}

/**
* Constructor from KeyValue
* Constructor from KeyValue This avoids copying the value from the cell, and tries to optimize
* generating the column value.
*/
public CellModel(org.apache.hadoop.hbase.Cell cell) {
this(CellUtil.cloneFamily(cell), CellUtil.cloneQualifier(cell), cell.getTimestamp(),
CellUtil.cloneValue(cell));
this.column = makeColumn(cell.getFamilyArray(), cell.getFamilyOffset(), cell.getFamilyLength(),
cell.getQualifierArray(), cell.getQualifierOffset(), cell.getQualifierLength());
this.timestamp = cell.getTimestamp();
this.value = cell.getValueArray();
this.valueOffset = cell.getValueOffset();
this.valueLength = cell.getValueLength();
}

/**
Expand All @@ -107,16 +123,16 @@ public CellModel(org.apache.hadoop.hbase.Cell cell) {
public CellModel(byte[] column, long timestamp, byte[] value) {
this.column = column;
this.timestamp = timestamp;
this.value = value;
setValue(value);
}

/**
* Constructor
*/
public CellModel(byte[] column, byte[] qualifier, long timestamp, byte[] value) {
this.column = CellUtil.makeColumn(column, qualifier);
public CellModel(byte[] family, byte[] qualifier, long timestamp, byte[] value) {
this.column = CellUtil.makeColumn(family, qualifier);
this.timestamp = timestamp;
this.value = value;
setValue(value);
}

/** Returns the column */
Expand Down Expand Up @@ -149,22 +165,49 @@ public void setTimestamp(long timestamp) {
}

/** Returns the value */
@JsonProperty("$")
@XmlValue
public byte[] getValue() {
if (valueLength == MAGIC_LENGTH) {
return value;
} else {
byte[] retValue = new byte[valueLength];
System.arraycopy(value, valueOffset, retValue, 0, valueLength);
return retValue;
}
}

/** Returns the backing array for value (may be the same as value) */
public byte[] getValueArray() {
return value;
}

/**
* @param value the value to set
*/
@JsonProperty("$")
public void setValue(byte[] value) {
this.value = value;
this.valueLength = MAGIC_LENGTH;
}

public int getValueOffset() {
return valueOffset;
}

public int getValueLength() {
return valueLength;
}

@Override
public byte[] createProtobufOutput() {
Cell.Builder builder = Cell.newBuilder();
builder.setColumn(ByteStringer.wrap(getColumn()));
builder.setData(ByteStringer.wrap(getValue()));
if (valueLength == MAGIC_LENGTH) {
builder.setData(ByteStringer.wrap(getValue()));
} else {
builder.setData(ByteStringer.wrap(value, valueOffset, valueLength));
}
if (hasUserTimestamp()) {
builder.setTimestamp(getTimestamp());
}
Expand All @@ -183,6 +226,21 @@ public ProtobufMessageHandler getObjectFromMessage(byte[] message) throws IOExce
return this;
}

/**
* Makes a column in family:qualifier form from separate byte arrays with offset and length.
* <p>
* Not recommended for usage as this is old-style API.
* @return family:qualifier
*/
public static byte[] makeColumn(byte[] family, int familyOffset, int familyLength,
byte[] qualifier, int qualifierOffset, int qualifierLength) {
byte[] column = new byte[familyLength + qualifierLength + 1];
System.arraycopy(family, familyOffset, column, 0, familyLength);
column[familyLength] = COLUMN_FAMILY_DELIMITER;
System.arraycopy(qualifier, qualifierOffset, column, familyLength + 1, qualifierLength);
return column;
}

@Override
public boolean equals(Object obj) {
if (obj == null) {
Expand All @@ -196,17 +254,17 @@ public boolean equals(Object obj) {
}
CellModel cellModel = (CellModel) obj;
return new EqualsBuilder().append(column, cellModel.column)
.append(timestamp, cellModel.timestamp).append(value, cellModel.value).isEquals();
.append(timestamp, cellModel.timestamp).append(getValue(), cellModel.getValue()).isEquals();
}

@Override
public int hashCode() {
return new HashCodeBuilder().append(column).append(timestamp).append(value).toHashCode();
return new HashCodeBuilder().append(column).append(timestamp).append(getValue()).toHashCode();
}

@Override
public String toString() {
return new ToStringBuilder(this).append("column", column).append("timestamp", timestamp)
.append("value", value).toString();
.append("value", getValue()).toString();
}
}
Loading

0 comments on commit fcfc112

Please sign in to comment.