Skip to content

Commit

Permalink
Add BlockTypeOperators
Browse files Browse the repository at this point in the history
  • Loading branch information
dain committed Oct 7, 2020
1 parent 46a9c57 commit 8ba235f
Show file tree
Hide file tree
Showing 8 changed files with 403 additions and 5 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -226,6 +226,7 @@
import io.prestosql.sql.analyzer.FeaturesConfig;
import io.prestosql.sql.tree.QualifiedName;
import io.prestosql.type.BigintOperators;
import io.prestosql.type.BlockTypeOperators;
import io.prestosql.type.BooleanOperators;
import io.prestosql.type.CharOperators;
import io.prestosql.type.DateOperators;
Expand Down Expand Up @@ -380,7 +381,11 @@ public class FunctionRegistry
private final Cache<FunctionBinding, WindowFunctionSupplier> specializedWindowCache;
private volatile FunctionMap functions = new FunctionMap();

public FunctionRegistry(Supplier<BlockEncodingSerde> blockEncodingSerdeSupplier, FeaturesConfig featuresConfig, TypeOperators typeOperators)
public FunctionRegistry(
Supplier<BlockEncodingSerde> blockEncodingSerdeSupplier,
FeaturesConfig featuresConfig,
TypeOperators typeOperators,
BlockTypeOperators blockTypeOperators)
{
// We have observed repeated compilation of MethodHandle that leads to full GCs.
// We notice that flushing the following caches mitigate the problem.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -106,6 +106,7 @@
import io.prestosql.sql.planner.PartitioningHandle;
import io.prestosql.sql.tree.QualifiedName;
import io.prestosql.transaction.TransactionManager;
import io.prestosql.type.BlockTypeOperators;
import io.prestosql.type.FunctionType;
import io.prestosql.type.InternalTypeManager;

Expand Down Expand Up @@ -206,10 +207,11 @@ public MetadataManager(
ColumnPropertyManager columnPropertyManager,
AnalyzePropertyManager analyzePropertyManager,
TransactionManager transactionManager,
TypeOperators typeOperators)
TypeOperators typeOperators,
BlockTypeOperators blockTypeOperators)
{
typeRegistry = new TypeRegistry(featuresConfig);
functions = new FunctionRegistry(this::getBlockEncodingSerde, featuresConfig, typeOperators);
functions = new FunctionRegistry(this::getBlockEncodingSerde, featuresConfig, typeOperators, blockTypeOperators);
functionResolver = new FunctionResolver(this);

this.procedures = new ProcedureRegistry(this);
Expand Down Expand Up @@ -274,7 +276,8 @@ public static MetadataManager createTestMetadataManager(TransactionManager trans
new ColumnPropertyManager(),
new AnalyzePropertyManager(),
transactionManager,
typeOperators);
typeOperators,
new BlockTypeOperators(typeOperators));
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -132,6 +132,7 @@
import io.prestosql.sql.planner.TypeAnalyzer;
import io.prestosql.sql.tree.Expression;
import io.prestosql.transaction.TransactionManagerConfig;
import io.prestosql.type.BlockTypeOperators;
import io.prestosql.type.TypeDeserializer;
import io.prestosql.type.TypeSignatureDeserializer;
import io.prestosql.type.TypeSignatureKeyDeserializer;
Expand Down Expand Up @@ -354,6 +355,7 @@ protected void setup(Binder binder)
binder.bind(MetadataManager.class).in(Scopes.SINGLETON);
binder.bind(Metadata.class).to(MetadataManager.class).in(Scopes.SINGLETON);
binder.bind(TypeOperators.class).in(Scopes.SINGLETON);
binder.bind(BlockTypeOperators.class).in(Scopes.SINGLETON);

// type
binder.bind(TypeAnalyzer.class).in(Scopes.SINGLETON);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -175,6 +175,7 @@
import io.prestosql.transaction.InMemoryTransactionManager;
import io.prestosql.transaction.TransactionManager;
import io.prestosql.transaction.TransactionManagerConfig;
import io.prestosql.type.BlockTypeOperators;
import io.prestosql.util.FinalizerService;
import org.intellij.lang.annotations.Language;
import org.weakref.jmx.MBeanExporter;
Expand Down Expand Up @@ -233,6 +234,7 @@ public class LocalQueryRunner
private final PlanFragmenter planFragmenter;
private final InMemoryNodeManager nodeManager;
private final TypeOperators typeOperators;
private final BlockTypeOperators blockTypeOperators;
private final MetadataManager metadata;
private final StatsCalculator statsCalculator;
private final CostCalculator costCalculator;
Expand Down Expand Up @@ -297,6 +299,7 @@ private LocalQueryRunner(
finalizerService.start();

this.typeOperators = new TypeOperators();
this.blockTypeOperators = new BlockTypeOperators(typeOperators);
this.sqlParser = new SqlParser();
this.nodeManager = new InMemoryNodeManager();
PageSorter pageSorter = new PagesIndexPageSorter(new PagesIndex.TestingFactory(false));
Expand All @@ -321,7 +324,8 @@ private LocalQueryRunner(
new ColumnPropertyManager(),
new AnalyzePropertyManager(),
transactionManager,
typeOperators);
typeOperators,
blockTypeOperators);
this.splitManager = new SplitManager(new QueryManagerConfig(), metadata);
this.planFragmenter = new PlanFragmenter(this.metadata, this.nodePartitioningManager, new QueryManagerConfig());
this.joinCompiler = new JoinCompiler(metadata);
Expand Down Expand Up @@ -483,6 +487,11 @@ public TypeOperators getTypeOperators()
return typeOperators;
}

public BlockTypeOperators getBlockTypeOperators()
{
return blockTypeOperators;
}

@Override
public NodePartitioningManager getNodePartitioningManager()
{
Expand Down
188 changes: 188 additions & 0 deletions presto-main/src/main/java/io/prestosql/type/BlockTypeOperators.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,188 @@
/*
* 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 io.prestosql.type;

import io.prestosql.spi.block.Block;
import io.prestosql.spi.function.InvocationConvention;
import io.prestosql.spi.type.Type;
import io.prestosql.spi.type.TypeOperators;

import javax.annotation.concurrent.GuardedBy;
import javax.inject.Inject;

import java.lang.invoke.MethodHandle;
import java.util.Objects;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ConcurrentMap;
import java.util.function.Supplier;

import static io.prestosql.spi.function.InvocationConvention.InvocationArgumentConvention.BLOCK_POSITION;
import static io.prestosql.spi.function.InvocationConvention.InvocationReturnConvention.FAIL_ON_NULL;
import static io.prestosql.spi.function.InvocationConvention.InvocationReturnConvention.NULLABLE_RETURN;
import static io.prestosql.spi.function.InvocationConvention.simpleConvention;
import static io.prestosql.type.SingleAccessMethodCompiler.compileSingleAccessMethod;
import static io.prestosql.type.TypeUtils.NULL_HASH_CODE;
import static java.util.Objects.requireNonNull;

public final class BlockTypeOperators
{
private static final InvocationConvention BLOCK_EQUAL_CONVENTION = simpleConvention(NULLABLE_RETURN, BLOCK_POSITION, BLOCK_POSITION);
private static final InvocationConvention HASH_CODE_CONVENTION = simpleConvention(FAIL_ON_NULL, BLOCK_POSITION);
private static final InvocationConvention XX_HASH_64_CONVENTION = simpleConvention(FAIL_ON_NULL, BLOCK_POSITION);
private static final InvocationConvention IS_DISTINCT_FROM_CONVENTION = simpleConvention(FAIL_ON_NULL, BLOCK_POSITION, BLOCK_POSITION);

private final ConcurrentMap<GeneratedBlockOperatorKey<?>, GeneratedBlockOperator<?>> generatedBlockOperatorCache = new ConcurrentHashMap<>();
private final TypeOperators typeOperators;

@Inject
public BlockTypeOperators(TypeOperators typeOperators)
{
this.typeOperators = requireNonNull(typeOperators, "typeOperators is null");
}

public BlockPositionEqual getEqualOperator(Type type)
{
return getBlockOperator(type, BlockPositionEqual.class, () -> typeOperators.getEqualOperator(type, BLOCK_EQUAL_CONVENTION));
}

public interface BlockPositionEqual
{
Boolean equal(Block left, int leftPosition, Block right, int rightPosition);

default boolean equalNullSafe(Block leftBlock, int leftPosition, Block rightBlock, int rightPosition)
{
boolean leftIsNull = leftBlock.isNull(leftPosition);
boolean rightIsNull = rightBlock.isNull(rightPosition);
if (leftIsNull || rightIsNull) {
return leftIsNull && rightIsNull;
}
return equal(leftBlock, leftPosition, rightBlock, rightPosition);
}
}

public BlockPositionHashCode getHashCodeOperator(Type type)
{
return getBlockOperator(type, BlockPositionHashCode.class, () -> typeOperators.getHashCodeOperator(type, HASH_CODE_CONVENTION));
}

public interface BlockPositionHashCode
{
long hashCode(Block block, int position);

default long hashCodeNullSafe(Block block, int position)
{
if (block.isNull(position)) {
return NULL_HASH_CODE;
}
return hashCode(block, position);
}
}

public BlockPositionXxHash64 getXxHash64Operator(Type type)
{
return getBlockOperator(type, BlockPositionXxHash64.class, () -> typeOperators.getXxHash64Operator(type, XX_HASH_64_CONVENTION));
}

public interface BlockPositionXxHash64
{
long xxHash64(Block block, int position);
}

public BlockPositionIsDistinctFrom getDistinctFromOperator(Type type)
{
return getBlockOperator(type, BlockPositionIsDistinctFrom.class, () -> typeOperators.getDistinctFromOperator(type, IS_DISTINCT_FROM_CONVENTION));
}

public interface BlockPositionIsDistinctFrom
{
boolean isDistinctFrom(Block left, int leftPosition, Block right, int rightPosition);
}

private <T> T getBlockOperator(Type type, Class<T> operatorInterface, Supplier<MethodHandle> methodHandleSupplier)
{
@SuppressWarnings("unchecked")
GeneratedBlockOperator<T> generatedBlockOperator = (GeneratedBlockOperator<T>) generatedBlockOperatorCache.computeIfAbsent(
new GeneratedBlockOperatorKey<>(type, operatorInterface),
key -> new GeneratedBlockOperator<>(key.getType(), key.getOperatorInterface(), methodHandleSupplier.get()));
return generatedBlockOperator.get();
}

private static class GeneratedBlockOperatorKey<T>
{
private final Type type;
private final Class<T> operatorInterface;

public GeneratedBlockOperatorKey(Type type, Class<T> operatorInterface)
{
this.type = requireNonNull(type, "type is null");
this.operatorInterface = requireNonNull(operatorInterface, "operatorInterface is null");
}

public Type getType()
{
return type;
}

public Class<T> getOperatorInterface()
{
return operatorInterface;
}

@Override
public boolean equals(Object o)
{
if (this == o) {
return true;
}
if (o == null || getClass() != o.getClass()) {
return false;
}
GeneratedBlockOperatorKey<?> that = (GeneratedBlockOperatorKey<?>) o;
return type.equals(that.type) &&
operatorInterface.equals(that.operatorInterface);
}

@Override
public int hashCode()
{
return Objects.hash(type, operatorInterface);
}
}

private static class GeneratedBlockOperator<T>
{
private final Type type;
private final Class<T> operatorInterface;
private final MethodHandle methodHandle;
@GuardedBy("this")
private T operator;

public GeneratedBlockOperator(Type type, Class<T> operatorInterface, MethodHandle methodHandle)
{
this.type = requireNonNull(type, "type is null");
this.operatorInterface = requireNonNull(operatorInterface, "operatorInterface is null");
this.methodHandle = requireNonNull(methodHandle, "methodHandle is null");
}

public synchronized T get()
{
if (operator != null) {
return operator;
}
String suggestedClassName = operatorInterface.getSimpleName() + "_" + type.getDisplayName();
operator = compileSingleAccessMethod(suggestedClassName, operatorInterface, methodHandle);
return operator;
}
}
}
Loading

0 comments on commit 8ba235f

Please sign in to comment.