Skip to content

Commit

Permalink
Move Block.read and BlockBuilder.write to TypeUtils
Browse files Browse the repository at this point in the history
  • Loading branch information
haozhun committed Nov 20, 2015
1 parent 7f77a9a commit 64b2b7d
Show file tree
Hide file tree
Showing 9 changed files with 106 additions and 73 deletions.
Expand Up @@ -24,6 +24,7 @@
import java.util.List;
import java.util.Map;

import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;
import static com.google.common.base.Preconditions.checkArgument;
import static java.util.Objects.requireNonNull;

Expand Down Expand Up @@ -96,7 +97,7 @@ public Builder add(Object... values)
{
pageBuilder.declarePosition();
for (int i = 0; i < types.size(); i++) {
pageBuilder.getBlockBuilder(i).write(types.get(i), values[i]);
writeNativeValue(types.get(i), pageBuilder.getBlockBuilder(i), values[i]);
}

if (pageBuilder.isFull()) {
Expand Down
Expand Up @@ -47,6 +47,7 @@
import java.util.concurrent.ConcurrentMap;

import static com.facebook.presto.spi.StandardErrorCode.INVALID_SESSION_PROPERTY;
import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;
import static com.facebook.presto.sql.planner.ExpressionInterpreter.evaluateConstantExpression;
import static com.google.common.base.MoreObjects.firstNonNull;
import static com.google.common.base.Preconditions.checkArgument;
Expand Down Expand Up @@ -193,7 +194,7 @@ public static Object evaluatePropertyValue(Expression expression, Type expectedT

// convert to object value type of SQL type
BlockBuilder blockBuilder = expectedType.createBlockBuilder(new BlockBuilderStatus(), 1);
blockBuilder.write(expectedType, value);
writeNativeValue(expectedType, blockBuilder, value);
Object objectValue = expectedType.getObjectValue(session.toConnectorSession(), blockBuilder, 0);

if (objectValue == null) {
Expand Down
Expand Up @@ -32,6 +32,7 @@

import static com.facebook.presto.spi.StandardErrorCode.INVALID_TABLE_PROPERTY;
import static com.facebook.presto.spi.StandardErrorCode.NOT_FOUND;
import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;
import static com.facebook.presto.sql.planner.ExpressionInterpreter.evaluateConstantExpression;
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Preconditions.checkState;
Expand Down Expand Up @@ -121,7 +122,7 @@ private static Object evaluatePropertyValue(Expression expression, Type expected

// convert to object value type of SQL type
BlockBuilder blockBuilder = expectedType.createBlockBuilder(new BlockBuilderStatus(), 1);
blockBuilder.write(expectedType, value);
writeNativeValue(expectedType, blockBuilder, value);
Object objectValue = expectedType.getObjectValue(session.toConnectorSession(), blockBuilder, 0);

if (objectValue == null) {
Expand Down
Expand Up @@ -168,6 +168,7 @@
import static com.facebook.presto.operator.WindowFunctionDefinition.window;
import static com.facebook.presto.spi.StandardErrorCode.COMPILER_ERROR;
import static com.facebook.presto.spi.type.BigintType.BIGINT;
import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;
import static com.facebook.presto.sql.analyzer.ExpressionAnalyzer.getExpressionTypes;
import static com.facebook.presto.sql.analyzer.ExpressionAnalyzer.getExpressionTypesFromInput;
import static com.facebook.presto.sql.planner.plan.JoinNode.Type.FULL;
Expand Down Expand Up @@ -1033,7 +1034,7 @@ public PhysicalOperation visitValues(ValuesNode node, LocalExecutionPlanContext
for (int i = 0; i < row.size(); i++) {
// evaluate the literal value
Object result = ExpressionInterpreter.expressionInterpreter(row.get(i), metadata, context.getSession(), expressionTypes).evaluate(0);
pageBuilder.getBlockBuilder(i).write(outputTypes.get(i), result);
writeNativeValue(outputTypes.get(i), pageBuilder.getBlockBuilder(i), result);
}
}

Expand Down
Expand Up @@ -34,6 +34,7 @@
import static com.facebook.presto.spi.type.BigintType.BIGINT;
import static com.facebook.presto.spi.type.BooleanType.BOOLEAN;
import static com.facebook.presto.spi.type.DoubleType.DOUBLE;
import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;
import static com.facebook.presto.spi.type.VarcharType.VARCHAR;
import static org.testng.Assert.assertEquals;

Expand Down Expand Up @@ -192,8 +193,8 @@ private static void assertProjection(

private static Block createBlock(Type type, Object value)
{
return type.createBlockBuilder(new BlockBuilderStatus(), 1)
.write(type, value)
.build();
BlockBuilder blockBuilder = type.createBlockBuilder(new BlockBuilderStatus(), 1);
writeNativeValue(type, blockBuilder, value);
return blockBuilder.build();
}
}
24 changes: 0 additions & 24 deletions presto-spi/src/main/java/com/facebook/presto/spi/block/Block.java
Expand Up @@ -13,7 +13,6 @@
*/
package com.facebook.presto.spi.block;

import com.facebook.presto.spi.type.Type;
import io.airlift.slice.Slice;

import java.util.List;
Expand Down Expand Up @@ -68,29 +67,6 @@ default <T> T getObject(int position, Class<T> clazz)
throw new UnsupportedOperationException();
}

/**
* Gets the native value as an object in the value at {@code position}.
*/
default Object read(Type type, int position)
{
Class<?> javaType = type.getJavaType();
if (javaType == long.class) {
return type.getLong(this, position);
}
else if (javaType == double.class) {
return type.getDouble(this, position);
}
else if (javaType == boolean.class) {
return type.getBoolean(this, position);
}
else if (javaType == Slice.class) {
return type.getSlice(this, position);
}
else {
return type.getObject(this, position);
}
}

/**
* Is the byte sequences at {@code offset} in the value at {@code position} equal
* to the byte sequence at {@code otherOffset} in {@code otherSlice}.
Expand Down
Expand Up @@ -13,9 +13,7 @@
*/
package com.facebook.presto.spi.block;

import com.facebook.presto.spi.type.Type;
import io.airlift.slice.Slice;
import io.airlift.slice.Slices;

public interface BlockBuilder
extends Block
Expand Down Expand Up @@ -63,42 +61,6 @@ default BlockBuilder writeObject(Object value)
throw new UnsupportedOperationException();
}

/**
* Write a native value object to the current entry;
*/
default BlockBuilder write(Type type, Object value)
{
if (value == null) {
appendNull();
}
else if (type.getJavaType() == boolean.class) {
type.writeBoolean(this, (Boolean) value);
}
else if (type.getJavaType() == double.class) {
type.writeDouble(this, ((Number) value).doubleValue());
}
else if (type.getJavaType() == long.class) {
type.writeLong(this, ((Number) value).longValue());
}
else if (type.getJavaType() == Slice.class) {
Slice slice;
if (value instanceof byte[]) {
slice = Slices.wrappedBuffer((byte[]) value);
}
else if (value instanceof String) {
slice = Slices.utf8Slice((String) value);
}
else {
slice = (Slice) value;
}
type.writeSlice(this, slice, 0, slice.length());
}
else {
type.writeObject(this, value);
}
return this;
}

/**
* Return a writer to the current entry. The caller can operate on the returned caller to incrementally build the object. This is generally more efficient than
* building the object elsewhere and call writeObject afterwards because a large chunk of memory could potentially be unnecessarily copied in this process.
Expand Down
Expand Up @@ -14,9 +14,13 @@
package com.facebook.presto.spi.predicate;

import com.facebook.presto.spi.block.Block;
import com.facebook.presto.spi.block.BlockBuilder;
import com.facebook.presto.spi.block.BlockBuilderStatus;
import com.facebook.presto.spi.type.Type;

import static com.facebook.presto.spi.type.TypeUtils.readNativeValue;
import static com.facebook.presto.spi.type.TypeUtils.writeNativeValue;

public final class Utils
{
private Utils()
Expand All @@ -28,13 +32,13 @@ static Block nativeValueToBlock(Type type, Object object)
if (!Primitives.wrap(type.getJavaType()).isInstance(object)) {
throw new IllegalArgumentException(String.format("Object '%s' does not match type %s", object, type.getJavaType()));
}
return type.createBlockBuilder(new BlockBuilderStatus(), 1)
.write(type, object)
.build();
BlockBuilder blockBuilder = type.createBlockBuilder(new BlockBuilderStatus(), 1);
writeNativeValue(type, blockBuilder, object);
return blockBuilder.build();
}

static Object blockToNativeValue(Type type, Block block)
{
return block.read(type, 0);
return readNativeValue(type, block, 0);
}
}
@@ -0,0 +1,86 @@
/*
* Licensed 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 com.facebook.presto.spi.type;

import com.facebook.presto.spi.block.Block;
import com.facebook.presto.spi.block.BlockBuilder;
import io.airlift.slice.Slice;
import io.airlift.slice.Slices;

public final class TypeUtils
{
private TypeUtils()
{
}

/**
* Get the native value as an object in the value at {@code position} of {@code block}.
*/
public static Object readNativeValue(Type type, Block block, int position)
{
Class<?> javaType = type.getJavaType();

if (block.isNull(position)) {
return null;
}
if (javaType == long.class) {
return type.getLong(block, position);
}
if (javaType == double.class) {
return type.getDouble(block, position);
}
if (javaType == boolean.class) {
return type.getBoolean(block, position);
}
if (javaType == Slice.class) {
return type.getSlice(block, position);
}
return type.getObject(block, position);
}

/**
* Write a native value object to the current entry of {@code blockBuilder}.
*/
public static void writeNativeValue(Type type, BlockBuilder blockBuilder, Object value)
{
if (value == null) {
blockBuilder.appendNull();
}
else if (type.getJavaType() == boolean.class) {
type.writeBoolean(blockBuilder, (Boolean) value);
}
else if (type.getJavaType() == double.class) {
type.writeDouble(blockBuilder, ((Number) value).doubleValue());
}
else if (type.getJavaType() == long.class) {
type.writeLong(blockBuilder, ((Number) value).longValue());
}
else if (type.getJavaType() == Slice.class) {
Slice slice;
if (value instanceof byte[]) {
slice = Slices.wrappedBuffer((byte[]) value);
}
else if (value instanceof String) {
slice = Slices.utf8Slice((String) value);
}
else {
slice = (Slice) value;
}
type.writeSlice(blockBuilder, slice, 0, slice.length());
}
else {
type.writeObject(blockBuilder, value);
}
}
}

0 comments on commit 64b2b7d

Please sign in to comment.