You signed in with another tab or window. Reload to refresh your session.You signed out in another tab or window. Reload to refresh your session.You switched accounts on another tab or window. Reload to refresh your session.Dismiss alert
// TODO: Load ChannelInstances from executionState? (as of now there is no input into PostgreSQL).
/* * 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. */packageorg.apache.wayang.genericjdbc.execution;
importorg.apache.logging.log4j.LogManager;
importorg.apache.logging.log4j.Logger;
importorg.apache.wayang.basic.channels.FileChannel;
importorg.apache.wayang.core.api.Job;
importorg.apache.wayang.core.api.exception.WayangException;
importorg.apache.wayang.core.optimizer.OptimizationContext;
importorg.apache.wayang.core.plan.executionplan.Channel;
importorg.apache.wayang.core.plan.executionplan.ExecutionStage;
importorg.apache.wayang.core.plan.executionplan.ExecutionTask;
importorg.apache.wayang.core.plan.wayangplan.Operator;
importorg.apache.wayang.core.platform.ExecutionState;
importorg.apache.wayang.core.platform.Executor;
importorg.apache.wayang.core.platform.ExecutorTemplate;
importorg.apache.wayang.core.platform.Platform;
importorg.apache.wayang.core.util.WayangCollections;
importorg.apache.wayang.core.util.fs.FileSystem;
importorg.apache.wayang.core.util.fs.FileSystems;
importorg.apache.wayang.jdbc.channels.SqlQueryChannel;
importorg.apache.wayang.genericjdbc.operators.GenericJdbcExecutionOperator;
importorg.apache.wayang.genericjdbc.operators.GenericJdbcFilterOperator;
importorg.apache.wayang.genericjdbc.operators.GenericJdbcProjectionOperator;
importorg.apache.wayang.genericjdbc.operators.GenericJdbcTableSource;
importorg.apache.wayang.genericjdbc.platform.GenericJdbcPlatform;
importorg.apache.wayang.jdbc.compiler.FunctionCompiler;
importjava.io.IOException;
importjava.io.OutputStreamWriter;
importjava.io.UncheckedIOException;
importjava.sql.Connection;
importjava.sql.ResultSet;
importjava.sql.ResultSetMetaData;
importjava.sql.SQLException;
importjava.util.ArrayList;
importjava.util.Collection;
importjava.util.Set;
importjava.util.stream.Collectors;
/** * {@link Executor} implementation for the {@link GenericJdbcPlatform}. */publicclassGenericJdbcExecutorextendsExecutorTemplate {
privatefinalGenericJdbcPlatformplatform;
privatefinalConnectionconnection = null;
privatefinalLoggerlogger = LogManager.getLogger(this.getClass());
privatefinalFunctionCompilerfunctionCompiler = newFunctionCompiler();
publicGenericJdbcExecutor(GenericJdbcPlatformplatform, Jobjob) {
super(job.getCrossPlatformExecutor());
this.platform = platform;
// this.connection = this.platform.createDatabaseDescriptor(job.getConfiguration()).createJdbcConnection();
}
@Overridepublicvoidexecute(ExecutionStagestage, OptimizationContextoptimizationContext, ExecutionStateexecutionState) {
// TODO: Load ChannelInstances from executionState? (as of now there is no input into PostgreSQL).Collection<?> startTasks = stage.getStartTasks();
Collection<?> termTasks = stage.getTerminalTasks();
// Verify that we can handle this instance.assertstartTasks.size() == 1 : "Invalid jdbc stage: multiple sources are not currently supported";
ExecutionTaskstartTask = (ExecutionTask) startTasks.toArray()[0];
asserttermTasks.size() == 1 : "Invalid JDBC stage: multiple terminal tasks are not currently supported.";
ExecutionTasktermTask = (ExecutionTask) termTasks.toArray()[0];
assertstartTask.getOperator() instanceofGenericJdbcTableSource : "Invalid JDBC stage: Start task has to be a TableSource";
// Extract the different types of ExecutionOperators from the stage.GenericJdbcTableSourcetableOp = (GenericJdbcTableSource) startTask.getOperator();
SqlQueryChannel.InstancetipChannelInstance = this.instantiateOutboundChannel(startTask, optimizationContext);
Collection<ExecutionTask> filterTasks = newArrayList<>(4);
ExecutionTaskprojectionTask = null;
Set<ExecutionTask> allTasks = stage.getAllTasks();
assertallTasks.size() <= 3;
ExecutionTasknextTask = this.findGenericJdbcExecutionOperatorTaskInStage(startTask, stage);
while (nextTask != null) {
// Evaluate the nextTask.if (nextTask.getOperator() instanceofGenericJdbcFilterOperator) {
filterTasks.add(nextTask);
} elseif (nextTask.getOperator() instanceofGenericJdbcProjectionOperator) {
assertprojectionTask == null; //Allow one projection operator per stage for now.projectionTask = nextTask;
} else {
thrownewWayangException(String.format("Unsupported JDBC execution task %s", nextTask.toString()));
}
// Move the tipChannelInstance.tipChannelInstance = this.instantiateOutboundChannel(nextTask, optimizationContext, tipChannelInstance);
// Go to the next nextTask.nextTask = this.findGenericJdbcExecutionOperatorTaskInStage(nextTask, stage);
}
// Create the SQL query.StringtableName = this.getSqlClause(tableOp);
StringjdbcName = tableOp.jdbcName;
Collection<String> conditions = filterTasks.stream()
.map(ExecutionTask::getOperator)
.map(this::getSqlClause)
.collect(Collectors.toList());
Stringprojection = projectionTask == null ? "*" : this.getSqlClause(projectionTask.getOperator());
Stringquery = this.createSqlQuery(tableName, conditions, projection);
tipChannelInstance.setSqlQuery(query);
tipChannelInstance.setJdbcName(jdbcName);
// Return the tipChannelInstance.executionState.register(tipChannelInstance);
}
/** * Retrieves the follow-up {@link ExecutionTask} of the given {@code task} unless it is not comprising a * {@link GenericJdbcExecutionOperator} and/or not in the given {@link ExecutionStage}. * * @param task whose follow-up {@link ExecutionTask} is requested; should have a single follower * @param stage in which the follow-up {@link ExecutionTask} should be * @return the said follow-up {@link ExecutionTask} or {@code null} if none */privateExecutionTaskfindGenericJdbcExecutionOperatorTaskInStage(ExecutionTasktask, ExecutionStagestage) {
asserttask.getNumOuputChannels() == 1;
finalChanneloutputChannel = task.getOutputChannel(0);
finalExecutionTaskconsumer = WayangCollections.getSingle(outputChannel.getConsumers());
returnconsumer.getStage() == stage && consumer.getOperator() instanceofGenericJdbcExecutionOperator ?
consumer :
null;
}
/** * Instantiates the outbound {@link SqlQueryChannel} of an {@link ExecutionTask}. * * @param task whose outbound {@link SqlQueryChannel} should be instantiated * @param optimizationContext provides information about the {@link ExecutionTask} * @return the {@link SqlQueryChannel.Instance} */privateSqlQueryChannel.InstanceinstantiateOutboundChannel(ExecutionTasktask,
OptimizationContextoptimizationContext) {
asserttask.getNumOuputChannels() == 1 : String.format("Illegal task: %s.", task);
asserttask.getOutputChannel(0) instanceofSqlQueryChannel : String.format("Illegal task: %s.", task);
SqlQueryChanneloutputChannel = (SqlQueryChannel) task.getOutputChannel(0);
OptimizationContext.OperatorContextoperatorContext = optimizationContext.getOperatorContext(task.getOperator());
returnoutputChannel.createInstance(this, operatorContext, 0);
}
/** * Instantiates the outbound {@link SqlQueryChannel} of an {@link ExecutionTask}. * * @param task whose outbound {@link SqlQueryChannel} should be instantiated * @param optimizationContext provides information about the {@link ExecutionTask} * @param predecessorChannelInstance preceeding {@link SqlQueryChannel.Instance} to keep track of lineage * @return the {@link SqlQueryChannel.Instance} */privateSqlQueryChannel.InstanceinstantiateOutboundChannel(ExecutionTasktask,
OptimizationContextoptimizationContext,
SqlQueryChannel.InstancepredecessorChannelInstance) {
finalSqlQueryChannel.InstancenewInstance = this.instantiateOutboundChannel(task, optimizationContext);
newInstance.getLineage().addPredecessor(predecessorChannelInstance.getLineage());
returnnewInstance;
}
/** * Creates a SQL query. * * @param tableName the table to be queried * @param conditions conditions for the {@code WHERE} clause * @param projection projection for the {@code SELECT} clause * @return the SQL query */protectedStringcreateSqlQuery(StringtableName, Collection<String> conditions, Stringprojection) {
StringBuildersb = newStringBuilder(1000);
sb.append("SELECT ").append(projection).append(" FROM ").append(tableName);
if (!conditions.isEmpty()) {
sb.append(" WHERE ");
Stringseparator = "";
for (Stringcondition : conditions) {
sb.append(separator).append(condition);
separator = " AND ";
}
}
sb.append(';');
returnsb.toString();
}
/** * Creates a SQL clause that corresponds to the given {@link Operator}. * * @param operator for that the SQL clause should be generated * @return the SQL clause */privateStringgetSqlClause(Operatoroperator) {
return ((GenericJdbcExecutionOperator) operator).createSqlClause(this.connection, this.functionCompiler);
}
@Overridepublicvoiddispose() {
// try {// this.connection.close();// } catch (SQLException e) {// this.logger.error("Could not close JDBC connection to PostgreSQL correctly.", e);// }return;
}
@OverridepublicPlatformgetPlatform() {
returnthis.platform;
}
privatevoidsaveResult(FileChannel.InstanceoutputFileChannelInstance, ResultSetrs) throwsIOException, SQLException {
// Output results.finalFileSystemoutFs = FileSystems.getFileSystem(outputFileChannelInstance.getSinglePath()).get();
try (finalOutputStreamWriterwriter = newOutputStreamWriter(outFs.create(outputFileChannelInstance.getSinglePath()))) {
while (rs.next()) {
//System.out.println(rs.getInt(1) + " " + rs.getString(2));ResultSetMetaDatarsmd = rs.getMetaData();
for (inti = 1; i <= rsmd.getColumnCount(); i++) {
writer.write(rs.getString(i));
if (i < rsmd.getColumnCount()) {
writer.write('\t');
}
}
if (!rs.isLast()) {
writer.write('\n');
}
}
} catch (UncheckedIOExceptione) {
throwe.getCause();
}
}
}
7a42fa55604535c1107f2822d76f2fccb089c23e
The text was updated successfully, but these errors were encountered:
Load ChannelInstances from executionState? (as of now there is no input into PostgreSQL).
this.connection.close();
} catch (SQLException e) {
this.logger.error("Could not close JDBC connection to PostgreSQL correctly.", e);
}
incubator-wayang/wayang-platforms/wayang-generic-jdbc/src/main/java/org/apache/wayang/genericjdbc/execution/GenericJdbcExecutor.java
Line 80 in 5e6a07e
7a42fa55604535c1107f2822d76f2fccb089c23e
The text was updated successfully, but these errors were encountered: