Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -24,18 +24,26 @@
import java.util.List;
import java.util.Objects;

import static com.facebook.presto.metadata.BuiltInFunctionKind.ENGINE;
import static java.util.Objects.requireNonNull;

public class BuiltInFunctionHandle
implements FunctionHandle
{
private final Signature signature;
private final BuiltInFunctionKind builtInFunctionKind;

@JsonCreator
public BuiltInFunctionHandle(@JsonProperty("signature") Signature signature)
{
this(signature, ENGINE);
}

public BuiltInFunctionHandle(Signature signature, BuiltInFunctionKind builtInFunctionKind)
{
this.signature = requireNonNull(signature, "signature is null");
checkArgument(signature.getTypeVariableConstraints().isEmpty(), "%s has unbound type parameters", signature);
this.builtInFunctionKind = requireNonNull(builtInFunctionKind, "builtInFunctionKind is null");
}

@JsonProperty
Expand Down Expand Up @@ -68,6 +76,12 @@ public CatalogSchemaName getCatalogSchemaName()
return signature.getName().getCatalogSchemaName();
}

@JsonProperty
public BuiltInFunctionKind getBuiltInFunctionKind()
{
return builtInFunctionKind;
}

@Override
public boolean equals(Object o)
{
Expand All @@ -78,13 +92,14 @@ public boolean equals(Object o)
return false;
}
BuiltInFunctionHandle that = (BuiltInFunctionHandle) o;
return Objects.equals(signature, that.signature);
return Objects.equals(signature, that.signature)
&& Objects.equals(builtInFunctionKind, that.builtInFunctionKind);
}

@Override
public int hashCode()
{
return Objects.hash(signature);
return Objects.hash(signature, builtInFunctionKind);
}

@Override
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,37 @@
/*
* 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.metadata;

import com.facebook.drift.annotations.ThriftEnum;
import com.facebook.drift.annotations.ThriftEnumValue;

@ThriftEnum
public enum BuiltInFunctionKind
{
ENGINE(0),
PLUGIN(1);

private final int value;

BuiltInFunctionKind(int value)
{
this.value = value;
}

@ThriftEnumValue
public int getValue()
{
return value;
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,259 @@
/*
* 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.metadata;

import com.facebook.presto.common.Page;
import com.facebook.presto.common.QualifiedObjectName;
import com.facebook.presto.common.block.BlockEncodingSerde;
import com.facebook.presto.common.function.SqlFunctionResult;
import com.facebook.presto.common.type.TypeManager;
import com.facebook.presto.common.type.UserDefinedType;
import com.facebook.presto.spi.PrestoException;
import com.facebook.presto.spi.function.AlterRoutineCharacteristics;
import com.facebook.presto.spi.function.FunctionHandle;
import com.facebook.presto.spi.function.FunctionMetadata;
import com.facebook.presto.spi.function.FunctionNamespaceManager;
import com.facebook.presto.spi.function.FunctionNamespaceTransactionHandle;
import com.facebook.presto.spi.function.Parameter;
import com.facebook.presto.spi.function.ScalarFunctionImplementation;
import com.facebook.presto.spi.function.Signature;
import com.facebook.presto.spi.function.SqlFunction;
import com.facebook.presto.spi.function.SqlInvokedFunction;
import com.facebook.presto.spi.function.SqlInvokedScalarFunctionImplementation;
import com.google.common.base.Supplier;
import com.google.common.base.Suppliers;
import com.google.common.cache.CacheBuilder;
import com.google.common.cache.CacheLoader;
import com.google.common.cache.LoadingCache;
import com.google.common.util.concurrent.UncheckedExecutionException;

import java.util.Collection;
import java.util.List;
import java.util.Optional;
import java.util.concurrent.CompletableFuture;

import static com.facebook.presto.metadata.BuiltInFunctionKind.PLUGIN;
import static com.facebook.presto.spi.function.FunctionImplementationType.SQL;
import static com.facebook.presto.spi.function.FunctionKind.SCALAR;
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Throwables.throwIfInstanceOf;
import static com.google.common.collect.ImmutableList.toImmutableList;
import static java.util.Collections.emptyList;
import static java.util.Objects.requireNonNull;
import static java.util.concurrent.TimeUnit.HOURS;

public class BuiltInPluginFunctionNamespaceManager
implements FunctionNamespaceManager<SqlFunction>
{
private volatile FunctionMap functions = new FunctionMap();
private final FunctionAndTypeManager functionAndTypeManager;
private final Supplier<FunctionMap> cachedFunctions =
Suppliers.memoize(this::checkForNamingConflicts);
private final LoadingCache<Signature, SpecializedFunctionKey> specializedFunctionKeyCache;
private final LoadingCache<SpecializedFunctionKey, ScalarFunctionImplementation> specializedScalarCache;

public BuiltInPluginFunctionNamespaceManager(FunctionAndTypeManager functionAndTypeManager)
{
this.functionAndTypeManager = requireNonNull(functionAndTypeManager, "functionAndTypeManager is null");
specializedFunctionKeyCache = CacheBuilder.newBuilder()
.maximumSize(1000)
.expireAfterWrite(1, HOURS)
.build(CacheLoader.from(this::doGetSpecializedFunctionKey));
specializedScalarCache = CacheBuilder.newBuilder()
.maximumSize(1000)
.expireAfterWrite(1, HOURS)
.build(CacheLoader.from(key -> {
checkArgument(
key.getFunction() instanceof SqlInvokedFunction,
"Unsupported scalar function class: %s",
key.getFunction().getClass());
return new SqlInvokedScalarFunctionImplementation(((SqlInvokedFunction) key.getFunction()).getBody());
}));
}

public synchronized void registerPluginFunctions(List<? extends SqlFunction> functions)
{
checkForNamingConflicts(functions);
this.functions = new FunctionMap(this.functions, functions);
}

@Override
public FunctionHandle getFunctionHandle(Optional<? extends FunctionNamespaceTransactionHandle> transactionHandle, Signature signature)
{
return new BuiltInFunctionHandle(signature, PLUGIN);
}

@Override
public Collection<SqlFunction> getFunctions(Optional<? extends FunctionNamespaceTransactionHandle> transactionHandle, QualifiedObjectName functionName)
{
if (functions.list().isEmpty() ||
(!functionName.getCatalogSchemaName().equals(functionAndTypeManager.getDefaultNamespace()))) {
return emptyList();
}
return cachedFunctions.get().get(functionName);
}

/**
* likePattern / escape is not used for optimization, returning all functions.
*/
@Override
public Collection<SqlFunction> listFunctions(Optional<String> likePattern, Optional<String> escape)
{
return cachedFunctions.get().list();
}

public FunctionMetadata getFunctionMetadata(FunctionHandle functionHandle)
{
checkArgument(functionHandle instanceof BuiltInFunctionHandle, "Expect BuiltInFunctionHandle");
Signature signature = ((BuiltInFunctionHandle) functionHandle).getSignature();
SpecializedFunctionKey functionKey;
try {
functionKey = specializedFunctionKeyCache.getUnchecked(signature);
}
catch (UncheckedExecutionException e) {
throwIfInstanceOf(e.getCause(), PrestoException.class);
throw e;
}
SqlFunction function = functionKey.getFunction();
checkArgument(function instanceof SqlInvokedFunction, "BuiltInPluginFunctionNamespaceManager only support SqlInvokedFunctions");
SqlInvokedFunction sqlFunction = (SqlInvokedFunction) function;
List<String> argumentNames = sqlFunction.getParameters().stream().map(Parameter::getName).collect(toImmutableList());
return new FunctionMetadata(
signature.getName(),
signature.getArgumentTypes(),
argumentNames,
signature.getReturnType(),
signature.getKind(),
sqlFunction.getRoutineCharacteristics().getLanguage(),
SQL,
function.isDeterministic(),
function.isCalledOnNullInput(),
sqlFunction.getVersion(),
sqlFunction.getComplexTypeFunctionDescriptor());
}

public ScalarFunctionImplementation getScalarFunctionImplementation(FunctionHandle functionHandle)
{
checkArgument(functionHandle instanceof BuiltInFunctionHandle, "Expect BuiltInFunctionHandle");
return getScalarFunctionImplementation(((BuiltInFunctionHandle) functionHandle).getSignature());
}

@Override
public void setBlockEncodingSerde(BlockEncodingSerde blockEncodingSerde)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support setting block encoding");
}

@Override
public FunctionNamespaceTransactionHandle beginTransaction()
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support setting block encoding");
}

@Override
public void commit(FunctionNamespaceTransactionHandle transactionHandle)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support setting block encoding");
}

@Override
public void abort(FunctionNamespaceTransactionHandle transactionHandle)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support setting block encoding");
}

@Override
public void createFunction(SqlInvokedFunction function, boolean replace)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support setting block encoding");
}

@Override
public void dropFunction(QualifiedObjectName functionName, Optional parameterTypes, boolean exists)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support drop function");
}

@Override
public void alterFunction(QualifiedObjectName functionName, Optional parameterTypes, AlterRoutineCharacteristics alterRoutineCharacteristics)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not alter function");
}

@Override
public void addUserDefinedType(UserDefinedType userDefinedType)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support adding user defined types");
}

@Override
public Optional<UserDefinedType> getUserDefinedType(QualifiedObjectName typeName)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not support getting user defined types");
}

@Override
public CompletableFuture<SqlFunctionResult> executeFunction(String source, FunctionHandle functionHandle, Page input, List channels, TypeManager typeManager)
{
throw new UnsupportedOperationException("BuiltInPluginFunctionNamespaceManager does not execute function");
}

private ScalarFunctionImplementation getScalarFunctionImplementation(Signature signature)
{
checkArgument(signature.getKind() == SCALAR, "%s is not a scalar function", signature);
checkArgument(signature.getTypeVariableConstraints().isEmpty(), "%s has unbound type parameters", signature);

try {
return specializedScalarCache.getUnchecked(getSpecializedFunctionKey(signature));
}
catch (UncheckedExecutionException e) {
throwIfInstanceOf(e.getCause(), PrestoException.class);
throw e;
}
}

private synchronized FunctionMap checkForNamingConflicts()
{
Optional<FunctionNamespaceManager<?>> functionNamespaceManager =
functionAndTypeManager.getServingFunctionNamespaceManager(functionAndTypeManager.getDefaultNamespace());
checkArgument(functionNamespaceManager.isPresent(), "Cannot find function namespace for catalog '%s'", functionAndTypeManager.getDefaultNamespace().getCatalogName());
checkForNamingConflicts(functionNamespaceManager.get().listFunctions(Optional.empty(), Optional.empty()));
return functions;
}

private synchronized void checkForNamingConflicts(Collection<? extends SqlFunction> functions)
{
for (SqlFunction function : functions) {
for (SqlFunction existingFunction : this.functions.list()) {
checkArgument(!function.getSignature().equals(existingFunction.getSignature()), "Function already registered: %s", function.getSignature());
}
}
}

private SpecializedFunctionKey doGetSpecializedFunctionKey(Signature signature)
{
return functionAndTypeManager.getSpecializedFunctionKey(signature, getFunctions(Optional.empty(), signature.getName()));
}

private SpecializedFunctionKey getSpecializedFunctionKey(Signature signature)
{
try {
return specializedFunctionKeyCache.getUnchecked(signature);
}
catch (UncheckedExecutionException e) {
throwIfInstanceOf(e.getCause(), PrestoException.class);
throw e;
}
}
}
Loading
Loading