[beam] branch master updated: [SQL] Support complex identifiers in DataCatalog
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 989c928 [SQL] Support complex identifiers in DataCatalog new fd67fd3 Merge pull request #9353 from akedin/datacatalog-custom-name-resolution 989c928 is described below commit 989c928f87b1e195d06862414b3ed6545e60ab2f Author: akedin AuthorDate: Thu Aug 15 12:09:39 2019 -0700 [SQL] Support complex identifiers in DataCatalog --- build.gradle | 1 + sdks/java/extensions/sql/datacatalog/build.gradle | 36 +++- .../datacatalog/DataCatalogTableProvider.java | 24 ++- .../meta/provider/datacatalog/ZetaSqlIdUtils.java | 79 .../datacatalog/DataCatalogBigQueryIT.java | 98 ++ .../provider/datacatalog/ZetaSqlIdUtilsTest.java | 64 +++ .../extensions/sql/impl/TableResolutionUtils.java | 4 +- .../sql/meta/provider/FullNameTableProvider.java | 166 + .../sql/meta/CustomTableResolverTest.java | 206 +++-- .../beam/sdk/io/gcp/bigquery/TestBigQuery.java | 4 + 10 files changed, 538 insertions(+), 144 deletions(-) diff --git a/build.gradle b/build.gradle index d7e7c06..fd0c74a 100644 --- a/build.gradle +++ b/build.gradle @@ -158,6 +158,7 @@ task javaPostCommit() { task sqlPostCommit() { dependsOn ":sdks:java:extensions:sql:postCommit" dependsOn ":sdks:java:extensions:sql:jdbc:postCommit" + dependsOn ":sdks:java:extensions:sql:datacatalog:postCommit" } task javaPostCommitPortabilityApi () { diff --git a/sdks/java/extensions/sql/datacatalog/build.gradle b/sdks/java/extensions/sql/datacatalog/build.gradle index b9bff77..530c2ec 100644 --- a/sdks/java/extensions/sql/datacatalog/build.gradle +++ b/sdks/java/extensions/sql/datacatalog/build.gradle @@ -1,5 +1,3 @@ -import groovy.json.JsonOutput - /* * Licensed to the Apache Software Foundation (ASF) under one * or more contributor license agreements. See the NOTICE file @@ -18,6 +16,8 @@ import groovy.json.JsonOutput * limitations under the License. */ +import groovy.json.JsonOutput + plugins { id 'org.apache.beam.module' } applyJavaNature() @@ -59,3 +59,35 @@ task runDataCatalogExample(type: JavaExec) { "--tempLocation=${gcsTempRoot}", ] } + + +task integrationTest(type: Test) { + group = "Verification" + def gcpProject = project.findProperty('gcpProject') ?: 'apache-beam-testing' + def gcsTempRoot = project.findProperty('gcsTempRoot') ?: 'gs://temp-storage-for-end-to-end-tests/' + + // Disable Gradle cache (it should not be used because the IT's won't run). + outputs.upToDateWhen { false } + + def pipelineOptions = [ + "--project=${gcpProject}", + "--tempLocation=${gcsTempRoot}", + "--blockOnRun=false"] + + systemProperty "beamTestPipelineOptions", JsonOutput.toJson(pipelineOptions) + + include '**/*IT.class' + maxParallelForks 4 + classpath = project(":sdks:java:extensions:sql:datacatalog") + .sourceSets + .test + .runtimeClasspath + testClassesDirs = files(project(":sdks:java:extensions:sql:datacatalog").sourceSets.test.output.classesDirs) + useJUnit {} +} + +task postCommit { + group = "Verification" + description = "Various integration tests" + dependsOn integrationTest +} diff --git a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java index 1ca0959..3fa8594 100644 --- a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java +++ b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java @@ -27,16 +27,19 @@ import java.util.Map; import java.util.stream.Stream; import javax.annotation.Nullable; import org.apache.beam.sdk.extensions.sql.BeamSqlTable; +import org.apache.beam.sdk.extensions.sql.impl.TableName; import org.apache.beam.sdk.extensions.sql.meta.Table; +import org.apache.beam.sdk.extensions.sql.meta.provider.FullNameTableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.TableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.bigquery.BigQueryTableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.pubsub.PubsubJsonTableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.text.TextTableProvider; import org.apache.
[beam] branch master updated: [SQL] Add custom table name resolution
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 2dc4667 [SQL] Add custom table name resolution new a44f7d8 Merge pull request #9343 from akedin/custom-table-name-resolution 2dc4667 is described below commit 2dc46671311b0203db5aaf2906e0ad8c21ed8b14 Author: akedin AuthorDate: Tue Apr 30 16:04:31 2019 -0700 [SQL] Add custom table name resolution --- .../extensions/sql/TableNameExtractionUtils.java | 98 .../extensions/sql/impl/CalciteQueryPlanner.java | 5 +- .../beam/sdk/extensions/sql/impl/TableName.java| 100 .../extensions/sql/impl/TableResolutionUtils.java | 214 + .../extensions/sql/meta/CustomTableResolver.java | 45 ++ .../CalciteCannotParseSimpleIdentifiersTest.java | 77 +++ .../sql/CalciteParsesSimpleIdentifiersTest.java| 133 ++ .../sql/meta/CustomTableResolverTest.java | 514 + 8 files changed, 1185 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/TableNameExtractionUtils.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/TableNameExtractionUtils.java new file mode 100644 index 000..556c246 --- /dev/null +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/TableNameExtractionUtils.java @@ -0,0 +1,98 @@ +/* + * 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.beam.sdk.extensions.sql; + +import static java.util.stream.Collectors.toList; + +import java.util.Collection; +import java.util.Collections; +import java.util.List; +import org.apache.beam.sdk.extensions.sql.impl.TableName; +import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableList; +import org.apache.calcite.sql.SqlAsOperator; +import org.apache.calcite.sql.SqlCall; +import org.apache.calcite.sql.SqlIdentifier; +import org.apache.calcite.sql.SqlJoin; +import org.apache.calcite.sql.SqlNode; +import org.apache.calcite.sql.SqlSelect; +import org.apache.calcite.sql.SqlSetOperator; + +/** + * Helper class to extract table identifiers from the query. + * + * Supports queries: + * + * + * ... FROM table... + * ... FROM table1, table2 AS x... + * ... FROM table1 JOIN (LEFT, INNER, OUTER etc) table2 JOIN table3 ... + * ... FROM table1 UNION (INTERSECT etc) SELECT ... + * + */ +public class TableNameExtractionUtils { + + public static List extractTableNamesFromNode(SqlNode node) { +if (node instanceof SqlSelect) { + return extractTableFromSelect((SqlSelect) node); +} + +if (node instanceof SqlIdentifier) { + return extractFromIdentifier((SqlIdentifier) node); +} + +if (node instanceof SqlJoin) { + return extractFromJoin((SqlJoin) node); +} + +if (node instanceof SqlCall) { + return extractFromCall((SqlCall) node); +} + +return Collections.emptyList(); + } + + private static List extractTableFromSelect(SqlSelect node) { +return extractTableNamesFromNode(node.getFrom()); + } + + private static List extractFromCall(SqlCall node) { +if (node.getOperator() instanceof SqlAsOperator) { + return extractTableNamesFromNode(node.getOperandList().get(0)); +} + +if (node.getOperator() instanceof SqlSetOperator) { + return node.getOperandList().stream() + .map(TableNameExtractionUtils::extractTableNamesFromNode) + .flatMap(Collection::stream) + .collect(toList()); +} + +return Collections.emptyList(); + } + + private static List extractFromJoin(SqlJoin join) { +return ImmutableList.builder() +.addAll(extractTableNamesFromNode(join.getLeft())) +.addAll(extractTableNamesFromNode(join.getRight())) +.build(); + } + + private static List extractFromIdentifier(SqlIdentifier identifier) { +return ImmutableList.of(TableName.create(identifier.names)); + } +} diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanne
[beam] branch master updated: Refactor release guide.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new d1cf9bb Refactor release guide. new b7bca84 Merge pull request #9319 from akedin/refactor-release-guide d1cf9bb is described below commit d1cf9bba61c3d35d84bdc4ebaf9bc1bd1ce9446c Author: akedin AuthorDate: Mon Aug 12 14:00:16 2019 -0700 Refactor release guide. Split the release guide into multiple steps, clean stale info up. --- website/src/_includes/section-menu/contribute.html | 17 +- website/src/contribute/release-guide.md| 1095 website/src/contribute/release-guide/1-prepare.md | 188 .../contribute/release-guide/2-create-branch.md| 132 +++ .../contribute/release-guide/3-verify-branch.md| 130 +++ .../contribute/release-guide/4-build-candidate.md | 148 +++ .../src/contribute/release-guide/5-triage-jira.md | 92 ++ .../src/contribute/release-guide/6-prepare-docs.md | 194 .../release-guide/7-validate-candidate.md | 147 +++ website/src/contribute/release-guide/8-finalize.md | 114 ++ .../src/contribute/release-guide/9-post-release.md | 59 ++ website/src/contribute/release-guide/index.md | 71 ++ 12 files changed, 1291 insertions(+), 1096 deletions(-) diff --git a/website/src/_includes/section-menu/contribute.html b/website/src/_includes/section-menu/contribute.html index 71c9f7d..d25a05e 100644 --- a/website/src/_includes/section-menu/contribute.html +++ b/website/src/_includes/section-menu/contribute.html @@ -38,6 +38,21 @@ Become a committer Committer guide -Release guide + + Release guide + + +Overview +Prepare +Create Branch +Verify Branch +Build Candidate +Triage JIRAs +Prepare Docs +Validate Candidate +Finalize +Post-release + + diff --git a/website/src/contribute/release-guide.md b/website/src/contribute/release-guide.md deleted file mode 100644 index 4a16fd1..000 --- a/website/src/contribute/release-guide.md +++ /dev/null @@ -1,1095 +0,0 @@ -layout: section -title: "Beam Release Guide" -section_menu: section-menu/contribute.html -permalink: /contribute/release-guide/ - - -# Apache Beam Release Guide - -* TOC -{:toc} - -## Introduction - -The Apache Beam project periodically declares and publishes releases. A release is one or more packages of the project artifact(s) that are approved for general public distribution and use. They may come with various degrees of caveat regarding their perceived quality and potential for change, such as “alpha”, “beta”, “incubating”, “stable”, etc. - -The Beam community treats releases with great importance. They are a public face of the project and most users interact with the project only through the releases. Releases are signed off by the entire Beam community in a public vote. - -Each release is executed by a *Release Manager*, who is selected among the Beam committers. This document describes the process that the Release Manager follows to perform a release. Any changes to this process should be discussed and adopted on the [dev@ mailing list]({{ site.baseurl }}/get-started/support/). - -Please remember that publishing software has legal consequences. This guide complements the foundation-wide [Product Release Policy](http://www.apache.org/dev/release.html) and [Release Distribution Policy](http://www.apache.org/dev/release-distribution). - -## Overview - -![Alt text]({{ "/images/release-guide-1.png" | prepend: site.baseurl }} "Release Process"){:width="100%"} - -The release process consists of several steps: - -1. Decide to release -1. Prepare for the release -1. Build a release candidate -1. Vote on the release candidate -1. During vote process, run validation tests -1. If necessary, fix any issues and go back to step 3. -1. Finalize the release -1. Promote the release - -** - -## Decide to release - -Deciding to release and selecting a Release Manager is the first step of the release process. This is a consensus-based decision of the entire community. - -Anybody can propose a release on the dev@ mailing list, giving a solid argument and nominating a committer as the Release Manager (including themselves). There’s no formal process, no vote requirements, and no timing requirements. Any objections should be resolved by consensus before starting the release. - -In general, the community prefers to have a rotating set of 3-5 Release Managers. Keeping a small core set of managers allows enough people to build expertise in this area and improve processes over time, without Release Managers needing to re-learn the processes for each release. That said, if you are a committer interested in s
[beam] branch master updated: [SQL] Disable KafkaCSVTableIT
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new f7eb6fe [SQL] Disable KafkaCSVTableIT new de18f4b Merge pull request #9322 from akedin/disable-kafkatableit f7eb6fe is described below commit f7eb6fe4eaddfdf393fb829f4eb920bdb5b2c979 Author: akedin AuthorDate: Mon Aug 12 15:10:52 2019 -0700 [SQL] Disable KafkaCSVTableIT --- .../beam/sdk/extensions/sql/meta/provider/kafka/KafkaCSVTableIT.java| 2 ++ 1 file changed, 2 insertions(+) diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/KafkaCSVTableIT.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/KafkaCSVTableIT.java index 201a1df..1d71621 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/KafkaCSVTableIT.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/KafkaCSVTableIT.java @@ -60,6 +60,7 @@ import org.apache.kafka.clients.producer.Producer; import org.apache.kafka.clients.producer.ProducerRecord; import org.junit.Assert; import org.junit.BeforeClass; +import org.junit.Ignore; import org.junit.Rule; import org.junit.Test; @@ -67,6 +68,7 @@ import org.junit.Test; * This is an integration test for KafkaCSVTable. There should be a kafka server running and the * address should be passed to it. (https://issues.apache.org/jira/projects/BEAM/issues/BEAM-7523) */ +@Ignore("https://issues.apache.org/jira/projects/BEAM/issues/BEAM-7523;) public class KafkaCSVTableIT { @Rule public transient TestPipeline pipeline = TestPipeline.create();
[beam] branch master updated: [BEAM-7896] Implementing RateEstimation for KafkaTable with Unit and Integration Tests
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 9d131d4 [BEAM-7896] Implementing RateEstimation for KafkaTable with Unit and Integration Tests new cd2ab9e Merge pull request #9298 from riazela/KafkaRateEstimation2 9d131d4 is described below commit 9d131d490dfa1b4838d0303a3f17f36202c0874b Author: Alireza Samadian AuthorDate: Tue Aug 6 16:56:03 2019 -0700 [BEAM-7896] Implementing RateEstimation for KafkaTable with Unit and Integration Tests --- sdks/java/extensions/sql/build.gradle | 1 + .../sql/meta/provider/kafka/BeamKafkaTable.java| 147 +-- .../meta/provider/kafka/BeamKafkaCSVTableTest.java | 118 - .../sql/meta/provider/kafka/KafkaCSVTableIT.java | 292 + .../sql/meta/provider/kafka/KafkaCSVTestTable.java | 197 ++ .../sql/meta/provider/kafka/KafkaTestRecord.java | 39 +++ 6 files changed, 777 insertions(+), 17 deletions(-) diff --git a/sdks/java/extensions/sql/build.gradle b/sdks/java/extensions/sql/build.gradle index b4a7079..fe07bfe 100644 --- a/sdks/java/extensions/sql/build.gradle +++ b/sdks/java/extensions/sql/build.gradle @@ -203,6 +203,7 @@ task integrationTest(type: Test) { systemProperty "beamTestPipelineOptions", JsonOutput.toJson(pipelineOptions) include '**/*IT.class' + exclude '**/KafkaCSVTableIT.java' maxParallelForks 4 classpath = project(":sdks:java:extensions:sql") .sourceSets diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java index 0e1dab3..11c12f6 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java @@ -19,9 +19,13 @@ package org.apache.beam.sdk.extensions.sql.meta.provider.kafka; import static org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Preconditions.checkArgument; +import java.util.Collection; import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.Properties; +import java.util.stream.Collectors; +import java.util.stream.Stream; import org.apache.beam.sdk.coders.ByteArrayCoder; import org.apache.beam.sdk.extensions.sql.impl.BeamTableStatistics; import org.apache.beam.sdk.extensions.sql.impl.schema.BaseBeamTable; @@ -34,9 +38,15 @@ import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.POutput; import org.apache.beam.sdk.values.Row; +import org.apache.kafka.clients.consumer.Consumer; +import org.apache.kafka.clients.consumer.ConsumerRecord; +import org.apache.kafka.clients.consumer.ConsumerRecords; +import org.apache.kafka.clients.consumer.KafkaConsumer; import org.apache.kafka.common.TopicPartition; import org.apache.kafka.common.serialization.ByteArrayDeserializer; import org.apache.kafka.common.serialization.ByteArraySerializer; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; /** * {@code BeamKafkaTable} represent a Kafka topic, as source or target. Need to extend to convert @@ -47,6 +57,10 @@ public abstract class BeamKafkaTable extends BaseBeamTable { private List topics; private List topicPartitions; private Map configUpdates; + private BeamTableStatistics rowCountStatistics = null; + private static final Logger LOGGER = LoggerFactory.getLogger(BeamKafkaTable.class); + // This is the number of records looked from each partition when the rate is estimated + protected int numberOfRecordsForRate = 50; protected BeamKafkaTable(Schema beamSchema) { super(beamSchema); @@ -84,7 +98,14 @@ public abstract class BeamKafkaTable extends BaseBeamTable { @Override public PCollection buildIOReader(PBegin begin) { -KafkaIO.Read kafkaRead = null; +return begin +.apply("read", createKafkaRead().withoutMetadata()) +.apply("in_format", getPTransformForInput()) +.setRowSchema(getSchema()); + } + + KafkaIO.Read createKafkaRead() { +KafkaIO.Read kafkaRead; if (topics != null) { kafkaRead = KafkaIO.read() @@ -104,28 +125,25 @@ public abstract class BeamKafkaTable extends BaseBeamTable { } else { throw new IllegalArgumentException("One of topics and topicPartitions must be configurated."); } - -return begin -.apply("read", kafkaRead.withoutMetadata()) -.apply("in_format", getPTransformForInput()) -.setRowSchema(getSchema()); +return kafk
[beam] branch release-2.15 created (now 0a2ddc0)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch release-2.15 in repository https://gitbox.apache.org/repos/asf/beam.git. at 0a2ddc0 Merge pull request #8859 from mszb/BEAM-7018 No new revisions were added by this update.
[beam] branch master updated (a5c8ae0 -> a694fda)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from a5c8ae0 Merge pull request #9245: [BEAM-7305] Adapt Jet Runner page to runner being released now new b47bc95 [BEAM-] Wiring up BeamCostModel new 5fb50c7 [BEAM-] Implementing beamComputeSelfCost for all the rel nodes new a694fda Merge pull request #9217 from riazela/BeamCostModel The 22589 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../extensions/sql/impl/CalciteQueryPlanner.java | 26 ++- .../extensions/sql/impl/planner/BeamCostModel.java | 253 + .../sql/impl/rel/BeamAggregationRel.java | 22 ++ .../sdk/extensions/sql/impl/rel/BeamCalcRel.java | 8 + .../sdk/extensions/sql/impl/rel/BeamIOSinkRel.java | 7 + .../extensions/sql/impl/rel/BeamIOSourceRel.java | 7 + .../extensions/sql/impl/rel/BeamIntersectRel.java | 14 ++ .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 11 + .../sdk/extensions/sql/impl/rel/BeamMinusRel.java | 13 ++ .../sdk/extensions/sql/impl/rel/BeamRelNode.java | 22 ++ .../sdk/extensions/sql/impl/rel/BeamSortRel.java | 12 + .../extensions/sql/impl/rel/BeamUncollectRel.java | 8 + .../sdk/extensions/sql/impl/rel/BeamUnionRel.java | 13 ++ .../sdk/extensions/sql/impl/rel/BeamUnnestRel.java | 8 + .../sdk/extensions/sql/impl/rel/BeamValuesRel.java | 8 + .../sql/impl/planner/BeamCostModelTest.java| 103 + .../sql/impl/planner/CalciteQueryPlannerTest.java | 73 ++ 17 files changed, 596 insertions(+), 12 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/BeamCostModel.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/planner/BeamCostModelTest.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/planner/CalciteQueryPlannerTest.java
[beam] branch master updated: [BEAM-7898] Remove default implementation of getRowCount and change the name to getTableStatistics
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 9cf6f0d [BEAM-7898] Remove default implementation of getRowCount and change the name to getTableStatistics new 0d911b8 Merge pull request #9254 from riazela/TablesStatEstimation 9cf6f0d is described below commit 9cf6f0d93fffda74c8075fe5858b17aadb8ad645 Author: Alireza Samadian AuthorDate: Mon Aug 5 11:11:56 2019 -0700 [BEAM-7898] Remove default implementation of getRowCount and change the name to getTableStatistics --- .../sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java | 7 +++ .../java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java| 9 + .../apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java| 2 +- .../apache/beam/sdk/extensions/sql/impl/BeamTableStatistics.java | 2 +- .../sdk/extensions/sql/impl/schema/BeamPCollectionTable.java | 7 +++ .../sdk/extensions/sql/meta/provider/bigquery/BigQueryTable.java | 6 +++--- .../sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java | 7 +++ .../sdk/extensions/sql/meta/provider/parquet/ParquetTable.java | 7 +++ .../extensions/sql/meta/provider/pubsub/PubsubIOJsonTable.java | 7 +++ .../sql/meta/provider/seqgen/GenerateSequenceTable.java | 7 +++ .../sdk/extensions/sql/meta/provider/test/TestBoundedTable.java | 2 +- .../sdk/extensions/sql/meta/provider/test/TestTableProvider.java | 2 +- .../extensions/sql/meta/provider/test/TestUnboundedTable.java| 2 +- .../beam/sdk/extensions/sql/meta/provider/text/TextTable.java| 4 ++-- .../sdk/extensions/sql/impl/rel/BeamEnumerableConverterTest.java | 6 ++ .../sql/impl/rel/BeamJoinRelUnboundedVsBoundedTest.java | 7 +++ .../beam/sdk/extensions/sql/impl/rule/JoinReorderingTest.java| 6 +++--- .../sql/meta/provider/bigquery/BigQueryRowCountIT.java | 6 +++--- .../extensions/sql/meta/provider/bigquery/BigQueryTestTable.java | 4 ++-- 19 files changed, 78 insertions(+), 22 deletions(-) diff --git a/sdks/java/extensions/sql/hcatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java b/sdks/java/extensions/sql/hcatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java index c96c657..2606941 100644 --- a/sdks/java/extensions/sql/hcatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java +++ b/sdks/java/extensions/sql/hcatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java @@ -21,8 +21,10 @@ import com.google.auto.value.AutoValue; import java.util.Map; import org.apache.beam.sdk.annotations.Experimental; import org.apache.beam.sdk.extensions.sql.BeamSqlTable; +import org.apache.beam.sdk.extensions.sql.impl.BeamTableStatistics; import org.apache.beam.sdk.io.hcatalog.HCatToRow; import org.apache.beam.sdk.io.hcatalog.HCatalogIO; +import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.schemas.Schema; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; @@ -68,6 +70,11 @@ public abstract class HCatalogTable implements BeamSqlTable { } @Override + public BeamTableStatistics getTableStatistics(PipelineOptions options) { +return BeamTableStatistics.BOUNDED_UNKNOWN; + } + + @Override public Schema getSchema() { return schema(); } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java index 6ddf8bd..ea7c030 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java @@ -39,8 +39,9 @@ public interface BeamSqlTable { /** Get the schema info of the table. */ Schema getSchema(); - /** Estimates the number of rows or the rate for unbounded Tables. */ - default BeamTableStatistics getRowCount(PipelineOptions options) { -return BeamTableStatistics.UNKNOWN; - } + /** + * Estimates the number of rows or the rate for unbounded Tables. If it is not possible to + * estimate the row count or rate it will return BeamTableStatistics.BOUNDED_UNKNOWN. + */ + BeamTableStatistics getTableStatistics(PipelineOptions options); } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java index b6dbf53..9a889a9 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java +++ b/sdks/java
[beam] branch master updated: [BEAM-7844] Implementing NodeStat Estimations for all the nodes
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 3ca9149 [BEAM-7844] Implementing NodeStat Estimations for all the nodes new 7d56a23 Merge pull request #9198 from riazela/RowRateWindowEstimation 3ca9149 is described below commit 3ca91490e0644ab89e5febc9d139402877f939e1 Author: Alireza Samadian AuthorDate: Thu Aug 1 09:02:11 2019 -0700 [BEAM-7844] Implementing NodeStat Estimations for all the nodes --- .../sql/impl/planner/NodeStatsMetadata.java| 4 +- .../sql/impl/rel/BeamAggregationRel.java | 39 - .../sdk/extensions/sql/impl/rel/BeamCalcRel.java | 22 - .../sdk/extensions/sql/impl/rel/BeamIOSinkRel.java | 2 +- .../extensions/sql/impl/rel/BeamIOSourceRel.java | 9 +- .../extensions/sql/impl/rel/BeamIntersectRel.java | 14 ++- .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 26 -- .../sdk/extensions/sql/impl/rel/BeamMinusRel.java | 8 +- .../sdk/extensions/sql/impl/rel/BeamSortRel.java | 3 +- .../extensions/sql/impl/rel/BeamSqlRelUtils.java | 19 + .../extensions/sql/impl/rel/BeamUncollectRel.java | 5 +- .../sdk/extensions/sql/impl/rel/BeamUnionRel.java | 10 ++- .../sdk/extensions/sql/impl/rel/BeamUnnestRel.java | 5 +- .../sdk/extensions/sql/impl/rel/BeamValuesRel.java | 2 +- .../extensions/sql/impl/planner/NodeStatsTest.java | 15 ...rceRelTest.java => BeamAggregationRelTest.java} | 70 ++- ...amIOSourceRelTest.java => BeamCalcRelTest.java} | 77 - .../sql/impl/rel/BeamIOSourceRelTest.java | 43 +- .../sql/impl/rel/BeamIntersectRelTest.java | 27 ++ .../impl/rel/BeamJoinRelBoundedVsBoundedTest.java | 75 .../rel/BeamJoinRelUnboundedVsBoundedTest.java | 41 + .../rel/BeamJoinRelUnboundedVsUnboundedTest.java | 50 ++- .../extensions/sql/impl/rel/BeamMinusRelTest.java | 99 +- .../extensions/sql/impl/rel/BeamSortRelTest.java | 39 +++-- .../sql/impl/rel/BeamUncollectRelTest.java | 47 ++ .../extensions/sql/impl/rel/BeamUnionRelTest.java | 53 .../extensions/sql/impl/rel/BeamValuesRelTest.java | 24 ++ 27 files changed, 745 insertions(+), 83 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStatsMetadata.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStatsMetadata.java index 4a9e79f..8bc62ee 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStatsMetadata.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStatsMetadata.java @@ -35,8 +35,8 @@ public interface NodeStatsMetadata extends Metadata { MetadataDef DEF = MetadataDef.of(NodeStatsMetadata.class, NodeStatsMetadata.Handler.class, METHOD); - // In order to use this we need to call it by relNode.metadata(RowRateWindowMetadata.class, - // mq).getRowRateWindow() where mq is the MetadataQuery (can be obtained by + // In order to use this we need to call it by relNode.metadata(NodeStatsMetadata.class, + // mq).getNodeStats() where mq is the MetadataQuery (can be obtained by // relNode.getCluster().getMetadataQuery()). After this, Calcite looks for the implementation of // this metadata that we have registered in MetadataProvider (it is RelMdNodeStats.class in // this case and we have registered it in CalciteQueryPlanner). Then Calcite's generated Code diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamAggregationRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamAggregationRel.java index 14e7475..4e5978e 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamAggregationRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamAggregationRel.java @@ -84,7 +84,44 @@ public class BeamAggregationRel extends Aggregate implements BeamRelNode { @Override public NodeStats estimateNodeStats(RelMetadataQuery mq) { -return NodeStats.create(mq.getRowCount(this)); + +NodeStats inputEstimate = BeamSqlRelUtils.getNodeStats(this.input, mq); + +inputEstimate = computeWindowingCostEffect(inputEstimate); + +NodeStats estimate; +// groupCount shows how many columns do we have in group by. One of them might be the windowing. +int groupCount = groupSet.cardinality() - (windowFn == null ? 0 : 1); +// This is similar to what Calcite does.If groupCount is zero then then we have only one value +// per window for unbounded and we have only one value for bounde
[beam] branch master updated: Fix date on 2.14.0 download links
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 48326ba Fix date on 2.14.0 download links new b14bbd4 Merge pull request #9226 from akedin/fix-214-date 48326ba is described below commit 48326ba4905f3c5ac433edcd56bb2879f5facfe7 Author: Anton Kedin AuthorDate: Thu Aug 1 15:35:40 2019 -0700 Fix date on 2.14.0 download links --- website/src/_posts/2019-07-31-beam-2.14.0.md | 2 +- website/src/get-started/downloads.md | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) diff --git a/website/src/_posts/2019-07-31-beam-2.14.0.md b/website/src/_posts/2019-07-31-beam-2.14.0.md index b35406d..1f8db7f 100644 --- a/website/src/_posts/2019-07-31-beam-2.14.0.md +++ b/website/src/_posts/2019-07-31-beam-2.14.0.md @@ -26,7 +26,7 @@ limitations under the License. --> We are happy to present the new 2.14.0 release of Beam. This release includes both improvements and new functionality. -See the [download page]({{ site.baseurl }}/get-started/downloads/#2140-2019-07-25) for this release. +See the [download page]({{ site.baseurl }}/get-started/downloads/#2140-2019-08-01) for this release. For more information on changes in 2.14.0, check out the [detailed release notes](https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12319527=12345431). diff --git a/website/src/get-started/downloads.md b/website/src/get-started/downloads.md index 80b9850..2ac4b1f 100644 --- a/website/src/get-started/downloads.md +++ b/website/src/get-started/downloads.md @@ -90,7 +90,7 @@ versions denoted `0.x.y`. ## Releases -## 2.14.0 (2019-07-25) +## 2.14.0 (2019-08-01) Official [source code download](http://www.apache.org/dyn/closer.cgi/beam/2.14.0/apache-beam-2.14.0-source-release.zip). [SHA-512](https://www.apache.org/dist/beam/2.14.0/apache-beam-2.14.0-source-release.zip.sha512). [signature](https://www.apache.org/dist/beam/2.14.0/apache-beam-2.14.0-source-release.zip.asc).
[beam] annotated tag v2.14.0 updated (ae926f9 -> acd419b)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to annotated tag v2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git. *** WARNING: tag v2.14.0 was modified! *** from ae926f9 (commit) to acd419b (tag) tagging ae926f9c751248e9504fd542880040205bebb096 (commit) replaces java-ulr-removal by Anton Kedin on Thu Aug 1 16:02:40 2019 -0700 - Log - Apache Beam 2.14.0 Release -BEGIN PGP SIGNATURE- iQIzBAABCgAdFiEEieL/yufpnPbmgnz+9zSfIxD/sZMFAl1Db5wACgkQ9zSfIxD/ sZOgzQ/+InIYSVovszf6djM92YkaWYxC5TWlLhC7GE8uviXWfKCC4SH7fy4uhFgu iWknfjMdQRst+QjgisX3rIKswTLFCjLZBKUXj3er4LQd1cZMiQBwgBLmJwwWO8BQ xbizu1Vv6YxD2LV66hNLjS2FQex9yMdsYkQAvFmFl0hgBGnx5CQhoIlx8E5hM6CD wzJTg2dqakhQfeuf00Iu0Oa7SUKhcybecxgXakR/Fr5gCtJHPAwwUk3iYswByhOV uHM9O5yBcAmoyd6y6RSsLdR4Q4jPuInSnOvWxcj8pQYHwz7HPEQlzWrca/LR2l1x AS5n0cqvXfEqN7lMzDWtN57T1C7jasKpsjD5Dp5cqHqlavRXheY0tL8wXdF4DTNm QyfcnaRcuFFjD+5ZnGKtRtKgF83v0bjN5MEpeoFXeAKRpr+2ZCiRfK5uL7Cx/6Mu o/c4xDY/+2G0V+bCi7UgGV4HewNSJbQhCdH8r4e6NiKkc5HIg4/xBa2O190c6itd lYmGZEzkGTW+p1CzHXuItMTUcg2PLEifUkPLjDfauoiYh47M5w1g8dA/2pvEEIQ1 /lcTu7rQg/8v6R558IXHXMGwkJ3Cvwfw+n3qotpTqFTby77cs3bO60lCqGEImj7H SJypsWedIQJ6QDakzokr9r+UJQfRt33wEdhKjLj/9F05HKEwV/Q= =A27b -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
[beam] branch master updated (2129c31 -> a866d6c)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 2129c31 Merge pull request #9216 from ttanay/bq-copy-comment new a40a278 Added 2.14.0 blog post draft new bdf3ea7 fixup new a3ee812 fixup new 3a7d1db Add known issues new cab46a0 fixup new 22ad0c2 Updated the download page anchor new 1d2de66 Add ApproximateUnique link new a866d6c Merge pull request #9201 from aaltay/bl214 The 22523 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: website/src/_data/authors.yml| 3 + website/src/_posts/2019-07-31-beam-2.14.0.md | 106 +++ 2 files changed, 109 insertions(+) create mode 100644 website/src/_posts/2019-07-31-beam-2.14.0.md
[beam] branch master updated: [BEAM-7844] Custom MetadataHandler for NodeStats (RowRateWindow)
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 11f9ca5 [BEAM-7844] Custom MetadataHandler for NodeStats (RowRateWindow) new e5366eb Merge pull request #9185 from riazela/RowRateWindow 11f9ca5 is described below commit 11f9ca5adc656e06bcab168b23a3b6bc2ea09242 Author: Alireza Samadian AuthorDate: Mon Jul 29 12:54:16 2019 -0700 [BEAM-7844] Custom MetadataHandler for NodeStats (RowRateWindow) --- .../extensions/sql/impl/CalciteQueryPlanner.java | 5 +- .../sdk/extensions/sql/impl/planner/NodeStats.java | 86 ++ .../sql/impl/planner/NodeStatsMetadata.java| 55 ++ .../sql/impl/planner/RelMdNodeStats.java | 84 + .../sql/impl/rel/BeamAggregationRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamCalcRel.java | 6 ++ .../sdk/extensions/sql/impl/rel/BeamIOSinkRel.java | 7 ++ .../extensions/sql/impl/rel/BeamIOSourceRel.java | 6 ++ .../extensions/sql/impl/rel/BeamIntersectRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 6 ++ .../sdk/extensions/sql/impl/rel/BeamMinusRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamRelNode.java | 4 + .../sdk/extensions/sql/impl/rel/BeamSortRel.java | 7 ++ .../extensions/sql/impl/rel/BeamUncollectRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamUnionRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamUnnestRel.java | 7 ++ .../sdk/extensions/sql/impl/rel/BeamValuesRel.java | 7 ++ .../extensions/sql/impl/planner/NodeStatsTest.java | 79 18 files changed, 393 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java index 3d4e6ca..a4ec34f 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java @@ -17,6 +17,7 @@ */ package org.apache.beam.sdk.extensions.sql.impl; +import org.apache.beam.sdk.extensions.sql.impl.planner.RelMdNodeStats; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamLogicalConvention; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamRelNode; import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableList; @@ -152,7 +153,9 @@ class CalciteQueryPlanner implements QueryPlanner { .setMetadataProvider( ChainedRelMetadataProvider.of( ImmutableList.of( - NonCumulativeCostImpl.SOURCE, root.rel.getCluster().getMetadataProvider(; + NonCumulativeCostImpl.SOURCE, + RelMdNodeStats.SOURCE, + root.rel.getCluster().getMetadataProvider(; RelMetadataQuery.THREAD_PROVIDERS.set( JaninoRelMetadataProvider.of(root.rel.getCluster().getMetadataProvider())); root.rel.getCluster().invalidateMetadataQuery(); diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStats.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStats.java new file mode 100644 index 000..88d7ad2 --- /dev/null +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/planner/NodeStats.java @@ -0,0 +1,86 @@ +/* + * 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.beam.sdk.extensions.sql.impl.planner; + +import com.google.auto.value.AutoValue; + +/** This is a utility class to represent rowCount, rate and window. */ +@AutoValue +public abstract class NodeStats { + + /** + * Returns an instance with all values set to INFINITY. This will be only used when the node is + * not a BeamRelNode and we don't have an estimation implementation for it in the me
[beam] branch master updated: Update design-documents.md
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e641e27 Update design-documents.md new e4e4590 Merge pull request #9207 from riazela/patch-2 e641e27 is described below commit e641e27412300e8c043b426529659a9b1ec9c98a Author: Alireza SamadianZakaria AuthorDate: Wed Jul 31 10:55:45 2019 -0700 Update design-documents.md --- website/src/contribute/design-documents.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/website/src/contribute/design-documents.md b/website/src/contribute/design-documents.md index 36e2830..17ca78a 100644 --- a/website/src/contribute/design-documents.md +++ b/website/src/contribute/design-documents.md @@ -98,7 +98,7 @@ This is a collection of documents that may or may not be up to date. - Beam SQL Pipeline Options [[doc](https://docs.google.com/document/d/1UTsSBuruJRfGnVOS9eXbQI6NauCD4WnSAPgA_Y0zjdk)] - Unbounded limit [[doc](https://docs.google.com/document/d/13zeTewHH9nfwhSlcE4x77WQwr1U2Z4sTiNRjOXUj2aw)] - Portable Beam Schemas [[doc](https://s.apache.org/beam-schemas)] -- Cost Estimation [[doc](https://docs.google.com/document/d/1vi1PBBu5IqSy-qZl1Gk-49CcANOpbNs1UAud6LnOaiY/edit#heading=h.6rlkpwwx7gvf)] +- Cost Based Optimizer [[doc1](https://docs.google.com/document/d/1DM_bcfFbIoc_vEoqQxhC7AvHBUDVCAwToC8TYGukkII), [doc2](https://docs.google.com/document/d/1vi1PBBu5IqSy-qZl1Gk-49CcANOpbNs1UAud6LnOaiY/edit#heading=h.6rlkpwwx7gvf)] ### Portability - Fn API
[beam] branch master updated: [BEAM-7808] Add method to convert avro field to beam field
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 172b563 [BEAM-7808] Add method to convert avro field to beam field new 866b8e7 Merge pull request #9142 from bmv126/Avro_field_to_beam_field 172b563 is described below commit 172b563fdd36019d3284139417808681314ac364 Author: B M VISHWAS AuthorDate: Wed Jul 24 08:53:21 2019 -0500 [BEAM-7808] Add method to convert avro field to beam field --- .../apache/beam/sdk/schemas/utils/AvroUtils.java | 24 .../beam/sdk/schemas/utils/AvroUtilsTest.java | 32 ++ 2 files changed, 50 insertions(+), 6 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/AvroUtils.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/AvroUtils.java index 81b834b..0933b31 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/AvroUtils.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/AvroUtils.java @@ -163,6 +163,22 @@ public class AvroUtils { } } + /** Get Beam Field from avro Field. */ + public static Schema.Field toBeamField(org.apache.avro.Schema.Field field) { +TypeWithNullability nullableType = new TypeWithNullability(field.schema()); +FieldType beamFieldType = toFieldType(nullableType); +return Field.of(field.name(), beamFieldType); + } + + /** Get Avro Field from Beam Field. */ + public static org.apache.avro.Schema.Field toAvroField(Schema.Field field) { +org.apache.avro.Schema fieldSchema = getFieldSchema(field.getType()); +org.apache.avro.Schema.Field avroField = +new org.apache.avro.Schema.Field( +field.getName(), fieldSchema, field.getDescription(), (Object) null); +return avroField; + } + private AvroUtils() {} /** @@ -174,8 +190,7 @@ public class AvroUtils { Schema.Builder builder = Schema.builder(); for (org.apache.avro.Schema.Field field : schema.getFields()) { - TypeWithNullability nullableType = new TypeWithNullability(field.schema()); - Field beamField = Field.of(field.name(), toFieldType(nullableType)); + Field beamField = toBeamField(field); if (field.doc() != null) { beamField = beamField.withDescription(field.doc()); } @@ -189,10 +204,7 @@ public class AvroUtils { public static org.apache.avro.Schema toAvroSchema(Schema beamSchema) { List fields = Lists.newArrayList(); for (Schema.Field field : beamSchema.getFields()) { - org.apache.avro.Schema fieldSchema = getFieldSchema(field.getType()); - org.apache.avro.Schema.Field recordField = - new org.apache.avro.Schema.Field( - field.getName(), fieldSchema, field.getDescription(), (Object) null); + org.apache.avro.Schema.Field recordField = toAvroField(field); fields.add(recordField); } org.apache.avro.Schema avroSchema = org.apache.avro.Schema.createRecord(fields); diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/AvroUtilsTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/AvroUtilsTest.java index 94ab0d7..e43985c 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/AvroUtilsTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/AvroUtilsTest.java @@ -140,6 +140,38 @@ public class AvroUtilsTest { typeWithNullability.type); } + @Test + public void testNullableArrayFieldToBeamArrayField() { +org.apache.avro.Schema.Field avroField = +new org.apache.avro.Schema.Field( +"arrayField", +ReflectData.makeNullable( + org.apache.avro.Schema.createArray((org.apache.avro.Schema.create(Type.INT, +"", +null); + +Field expectedBeamField = Field.nullable("arrayField", FieldType.array(FieldType.INT32)); + +Field beamField = AvroUtils.toBeamField(avroField); +assertEquals(expectedBeamField, beamField); + } + + @Test + public void testNullableBeamArrayFieldToAvroField() { +Field beamField = Field.nullable("arrayField", FieldType.array(FieldType.INT32)); + +org.apache.avro.Schema.Field expectedAvroField = +new org.apache.avro.Schema.Field( +"arrayField", +ReflectData.makeNullable( + org.apache.avro.Schema.createArray((org.apache.avro.Schema.create(Type.INT, +"", +null); + +org.apache.avro.Schema.Field avroField = AvroUtils.toAvroField(beamField); +assertEquals(expectedAvroField, avroField); + } + private org.apache.avro.Schema getAvroSubSchema() { List fields = Lists.newArrayList(); fields.add(
[beam] branch master updated: [BEAM-7623] Add support to select MAP with Row as values in Beam SQL
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e813673 [BEAM-7623] Add support to select MAP with Row as values in Beam SQL new c80ca6b Merge pull request #9181 from bmv126/map_with_row_as_value e813673 is described below commit e81367329b103b255931d95fe60a5cb6e089b695 Author: B M VISHWAS AuthorDate: Mon Jul 29 07:26:14 2019 -0500 [BEAM-7623] Add support to select MAP with Row as values in Beam SQL --- .../sdk/extensions/sql/impl/rel/BeamCalcRel.java | 8 +++ .../sdk/extensions/sql/BeamComplexTypeTest.java| 79 ++ 2 files changed, 87 insertions(+) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java index de54625..78c8939 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java @@ -45,6 +45,7 @@ import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.PCollectionList; import org.apache.beam.sdk.values.Row; import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableMap; +import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.Maps; import org.apache.calcite.DataContext; import org.apache.calcite.adapter.enumerable.JavaRowFormat; import org.apache.calcite.adapter.enumerable.PhysType; @@ -418,6 +419,9 @@ public class BeamCalcRel extends Calc implements BeamRelNode { Expressions.equal(field, Expressions.constant(null)), Expressions.constant(null), Expressions.call(WrappedList.class, "of", field)); + } else if (fromType.getTypeName().isMapType() + && fromType.getMapValueType().getTypeName().isCompositeType()) { +field = nullOr(field, Expressions.call(WrappedList.class, "ofMapValues", field)); } else if (fromType.getTypeName() == TypeName.BYTES) { field = Expressions.condition( @@ -486,6 +490,10 @@ public class BeamCalcRel extends Calc implements BeamRelNode { return new WrappedList(row.getValues()); } +public static Map ofMapValues(Map map) { + return Maps.transformValues(map, val -> (val == null) ? null : WrappedList.of(val)); +} + @Override public Object get(int index) { Object obj = list.get(index); diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java index f75e92d..40caf5c 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java @@ -18,6 +18,8 @@ package org.apache.beam.sdk.extensions.sql; import java.util.Arrays; +import java.util.HashMap; +import java.util.Map; import org.apache.beam.sdk.extensions.sql.impl.BeamSqlEnv; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamSqlRelUtils; import org.apache.beam.sdk.extensions.sql.meta.provider.ReadOnlyTableProvider; @@ -430,4 +432,81 @@ public class BeamComplexTypeTest { pipeline.run().waitUntilFinish(Duration.standardMinutes(2)); } + + @Test + public void testMapWithRowAsValue() { + +Schema inputSchema = +Schema.builder() +.addMapField("mapWithValueAsRow", FieldType.STRING, FieldType.row(rowWithArraySchema)) +.build(); + +Map mapWithValueAsRow = new HashMap<>(); +Row complexRow = +Row.withSchema(rowWithArraySchema) +.addValues("RED", 5L, Arrays.asList(10L, 20L, 30L)) +.build(); +mapWithValueAsRow.put("key", complexRow); + +Row rowOfMap = Row.withSchema(inputSchema).addValue(mapWithValueAsRow).build(); + +PCollection outputRow = +pipeline +.apply(Create.of(rowOfMap)) +.setRowSchema(inputSchema) +.apply( +SqlTransform.query( +"select PCOLLECTION.mapWithValueAsRow['key'].field1 as color, PCOLLECTION.mapWithValueAsRow['key'].field3[2] as num from PCOLLECTION")); + +Row expectedRow = + Row.withSchema(Schema.builder().addStringField("color").addInt64Field("num").build()) +.addValues("RED", 20L) +.build(); + +PAssert.that(outputRow).containsInAnyOrder(expectedRow); +pipeline.run().waitUntilFinish(Duration.standar
[beam] branch master updated: [BEAM-7783] BeamIOSourceRel using custom BeamStatistics
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new abe5a7e [BEAM-7783] BeamIOSourceRel using custom BeamStatistics new e9e8731 Merge pull request #9161 from riazela/BeamTableStatistics abe5a7e is described below commit abe5a7e486f400d5c204e734482ba731a89f9921 Author: Alireza Samadian AuthorDate: Thu Jul 25 15:27:01 2019 -0700 [BEAM-7783] BeamIOSourceRel using custom BeamStatistics --- .../beam/sdk/extensions/sql/BeamSqlTable.java | 2 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 3 +- .../extensions/sql/impl/rel/BeamIOSourceRel.java | 18 ++- .../sql/meta/provider/test/TestBoundedTable.java | 7 ++ .../sql/meta/provider/test/TestUnboundedTable.java | 14 +++ .../sdk/extensions/sql/impl/rel/BaseRelTest.java | 6 +- .../sql/impl/rel/BeamIOSourceRelTest.java | 128 + .../rel/BeamJoinRelUnboundedVsBoundedTest.java | 2 +- 8 files changed, 174 insertions(+), 6 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java index b759761..6ddf8bd 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java @@ -39,7 +39,7 @@ public interface BeamSqlTable { /** Get the schema info of the table. */ Schema getSchema(); - /** Estimates the number of rows or returns null if there is no estimation. */ + /** Estimates the number of rows or the rate for unbounded Tables. */ default BeamTableStatistics getRowCount(PipelineOptions options) { return BeamTableStatistics.UNKNOWN; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java index 293a60b..b6dbf53 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteTable.java @@ -40,7 +40,6 @@ import org.apache.calcite.rel.type.RelDataTypeFactory; import org.apache.calcite.rex.RexNode; import org.apache.calcite.schema.ModifiableTable; import org.apache.calcite.schema.SchemaPlus; -import org.apache.calcite.schema.Statistic; import org.apache.calcite.schema.TranslatableTable; /** Adapter from {@link BeamSqlTable} to a calcite Table. */ @@ -81,7 +80,7 @@ public class BeamCalciteTable extends AbstractQueryableTable } @Override - public Statistic getStatistic() { + public BeamTableStatistics getStatistic() { /* Changing class loader is required for the JDBC path. It is similar to what done in {@link BeamEnumerableConverter#toRowList} and {@link BeamEnumerableConverter#toEnumerable }. diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamIOSourceRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamIOSourceRel.java index 82fcd3d..e22b64b 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamIOSourceRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamIOSourceRel.java @@ -22,11 +22,14 @@ import static org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Prec import java.util.Map; import org.apache.beam.sdk.extensions.sql.BeamSqlTable; import org.apache.beam.sdk.extensions.sql.impl.BeamCalciteTable; +import org.apache.beam.sdk.extensions.sql.impl.BeamTableStatistics; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.PCollectionList; import org.apache.beam.sdk.values.Row; import org.apache.calcite.plan.RelOptCluster; +import org.apache.calcite.plan.RelOptCost; +import org.apache.calcite.plan.RelOptPlanner; import org.apache.calcite.plan.RelOptTable; import org.apache.calcite.rel.core.TableScan; import org.apache.calcite.rel.metadata.RelMetadataQuery; @@ -52,7 +55,12 @@ public class BeamIOSourceRel extends TableScan implements BeamRelNode { @Override public double estimateRowCount(RelMetadataQuery mq) { -return super.estimateRowCount(mq); +BeamTableStatistics rowCountStatistics = calciteTable.getStatistic(); +if (beamTable.isBounded() == PCollection.IsBounded.BOUNDED) { + return rowCountStatistics.getRowCount(); +} else { + return rowCountStatistics.getRate(); +} } @Override @@ -78,6 +86,14 @@ public class
[beam] branch master updated: [BEAM-7823] Implementing a Custom NonCumulativeCost Handler
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 7a63558 [BEAM-7823] Implementing a Custom NonCumulativeCost Handler new 1c639d1 Merge pull request #9159 from riazela/CostMetadataHandler 7a63558 is described below commit 7a63558b113249c2773d2b16c29c615f60727412 Author: Alireza Samadian AuthorDate: Thu Jul 25 14:52:07 2019 -0700 [BEAM-7823] Implementing a Custom NonCumulativeCost Handler --- .../extensions/sql/impl/CalciteQueryPlanner.java | 62 +- 1 file changed, 61 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java index 3be2987..3d4e6ca 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteQueryPlanner.java @@ -24,12 +24,22 @@ import org.apache.calcite.config.CalciteConnectionConfig; import org.apache.calcite.jdbc.CalciteSchema; import org.apache.calcite.plan.Contexts; import org.apache.calcite.plan.ConventionTraitDef; +import org.apache.calcite.plan.RelOptCost; import org.apache.calcite.plan.RelOptPlanner.CannotPlanException; import org.apache.calcite.plan.RelOptUtil; import org.apache.calcite.plan.RelTraitDef; import org.apache.calcite.plan.RelTraitSet; import org.apache.calcite.prepare.CalciteCatalogReader; +import org.apache.calcite.rel.RelNode; import org.apache.calcite.rel.RelRoot; +import org.apache.calcite.rel.metadata.BuiltInMetadata; +import org.apache.calcite.rel.metadata.ChainedRelMetadataProvider; +import org.apache.calcite.rel.metadata.JaninoRelMetadataProvider; +import org.apache.calcite.rel.metadata.MetadataDef; +import org.apache.calcite.rel.metadata.MetadataHandler; +import org.apache.calcite.rel.metadata.ReflectiveRelMetadataProvider; +import org.apache.calcite.rel.metadata.RelMetadataProvider; +import org.apache.calcite.rel.metadata.RelMetadataQuery; import org.apache.calcite.schema.SchemaPlus; import org.apache.calcite.sql.SqlNode; import org.apache.calcite.sql.SqlOperatorTable; @@ -44,6 +54,7 @@ import org.apache.calcite.tools.Planner; import org.apache.calcite.tools.RelConversionException; import org.apache.calcite.tools.RuleSet; import org.apache.calcite.tools.ValidationException; +import org.apache.calcite.util.BuiltInMethod; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -135,8 +146,16 @@ class CalciteQueryPlanner implements QueryPlanner { .replace(BeamLogicalConvention.INSTANCE) .replace(root.collation) .simplify(); - // beam physical plan + root.rel + .getCluster() + .setMetadataProvider( + ChainedRelMetadataProvider.of( + ImmutableList.of( + NonCumulativeCostImpl.SOURCE, root.rel.getCluster().getMetadataProvider(; + RelMetadataQuery.THREAD_PROVIDERS.set( + JaninoRelMetadataProvider.of(root.rel.getCluster().getMetadataProvider())); + root.rel.getCluster().invalidateMetadataQuery(); beamRelNode = (BeamRelNode) planner.transform(0, desiredTraits, root.rel); LOG.info("BEAMPlan>\n" + RelOptUtil.toString(beamRelNode)); } catch (RelConversionException | CannotPlanException e) { @@ -149,4 +168,45 @@ class CalciteQueryPlanner implements QueryPlanner { } return beamRelNode; } + + // It needs to be public so that the generated code in Calcite can access it. + public static class NonCumulativeCostImpl + implements MetadataHandler { + +public static final RelMetadataProvider SOURCE = +ReflectiveRelMetadataProvider.reflectiveSource( +BuiltInMethod.NON_CUMULATIVE_COST.method, new NonCumulativeCostImpl()); + +@Override +public MetadataDef getDef() { + return BuiltInMetadata.NonCumulativeCost.DEF; +} + +@SuppressWarnings("UnusedDeclaration") +public RelOptCost getNonCumulativeCost(RelNode rel, RelMetadataQuery mq) { + // This is called by a generated code in calcite MetadataQuery. + // If the rel is Calcite rel or we are in JDBC path and cost factory is not set yet we should + // use calcite cost estimation + if (!(rel instanceof BeamRelNode)) { +return rel.computeSelfCost(rel.getCluster().getPlanner(), mq); + } + + // Currently we do nothing in this case, however, we can plug our own cost estimation method + // here and based on the design we also need to remove the cached values + + // We need to first remove the cached values. + +
[beam] tag v2.14.0-RC1 created (now ae926f9)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to tag v2.14.0-RC1 in repository https://gitbox.apache.org/repos/asf/beam.git. at ae926f9 (commit) No new revisions were added by this update.
svn commit: r35024 - /dev/beam/2.14.0/python/
Author: anton Date: Thu Jul 25 19:47:27 2019 New Revision: 35024 Log: Signed the wheels for Apache Beam 2.14 Added: dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_i686.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_i686.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_i686.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-manylinux1_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_i686.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_i686.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_i686.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27mu-manylinux1_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_i686.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_i686.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_i686.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp35-cp35m-manylinux1_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_i686.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_i686.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_i686.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp36-cp36m-manylinux1_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_i686.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_i686.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_i686.whl.sha512 dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_x86_64.whl (with props) dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_x86_64.whl.asc dev/beam/2.14.0/python/apache_beam-2.14.0-cp37-cp37m-manylinux1_x86_64.whl.sha512 Added: dev/beam/2.14.0/python/apache_beam-2.14.0-cp27-cp27m-macosx_10_6_intel.macosx_10_9_intel.macosx_10_9_x86_64.macosx_10_10_intel.macosx_10_10_x86_64.whl == Binary file - no diff
svn commit: r35012 - in /dev/beam/2.14.0/python: ./ apache-beam-2.14.0.zip apache-beam-2.14.0.zip.asc apache-beam-2.14.0.zip.sha512
Author: anton Date: Wed Jul 24 22:30:58 2019 New Revision: 35012 Log: Beam 2.14.0 release Added: dev/beam/2.14.0/python/ dev/beam/2.14.0/python/apache-beam-2.14.0.zip (with props) dev/beam/2.14.0/python/apache-beam-2.14.0.zip.asc dev/beam/2.14.0/python/apache-beam-2.14.0.zip.sha512 Added: dev/beam/2.14.0/python/apache-beam-2.14.0.zip == Binary file - no diff available. Propchange: dev/beam/2.14.0/python/apache-beam-2.14.0.zip -- svn:mime-type = application/octet-stream Added: dev/beam/2.14.0/python/apache-beam-2.14.0.zip.asc == --- dev/beam/2.14.0/python/apache-beam-2.14.0.zip.asc (added) +++ dev/beam/2.14.0/python/apache-beam-2.14.0.zip.asc Wed Jul 24 22:30:58 2019 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCgAdFiEEieL/yufpnPbmgnz+9zSfIxD/sZMFAl042+cACgkQ9zSfIxD/ +sZMz3BAA3NGLp1Tozxlm3GuVTTRQjrR52TI9eyUoGbSRBATIMq1vcWjLIX37YJWA +S5OWUyzRwtMD7zybIRv6VkmAcCoLunL5LxAhw+upRuOFryQOhcL1fd1/+0ywB0+k +EFUDZK+DwwiQkYKlmSxyA3FkiuLFadgAN5Ta6rZMNVNsFnRdIMKDIbiCHzftDCiJ +0eMemrhPolMNRh3J4CPOnrelIWhNfrVi5fWXVZ7gj9e8NDesYA5OE2WKUFxSH1sW +ZNh0IFqmukdhy70LWd0qs6bVOCn1FPkT0k8003te5x0fNHqSiMOO1rl05/lCNfx/ +F9VCZFwRLa534NQEerpgDSCaLstBMB//7nxmJ1TM8/guelB1r+ztX73wjuiA8WyE +g3XotwaNEHSXK58mZgz5BKy/yAoEYOGWvJcAP3hdXp7L1sna3az4S/lxxMKsChla +o+45JJnbyltMdQHb/WPny+mW0ZSbQ5+6EiGFuYmkPYz5+3ubCjgPgmPYFbdyC3Hb +EMzADTkiVsC/Al1dkL6SY+dzVX4yvf+vfWynAOrai8mGRV4Pfkdp+amj++yAXrsb +QRUl6J2Exqtm5vJHF4BRuNwaoariiowTbUyaBve16gfRlEjS+cn2PFMkOaZGPNyi +2jmnl3wfH5woydL0WKx2YgYpNQx8GtoUCHYQRq+9jnOMPSwyIXE= +=YU0C +-END PGP SIGNATURE- Added: dev/beam/2.14.0/python/apache-beam-2.14.0.zip.sha512 == --- dev/beam/2.14.0/python/apache-beam-2.14.0.zip.sha512 (added) +++ dev/beam/2.14.0/python/apache-beam-2.14.0.zip.sha512 Wed Jul 24 22:30:58 2019 @@ -0,0 +1 @@ +ebc7df230b48c3cc247af3a878f6f8a51ed2063db61bc93955bfd7359d788843fea4ed373c1bc2853490ee399a9a8b47c01ec0d30dd223d5996ab305e474c0c3 apache-beam-2.14.0.zip
svn commit: r35011 - in /dev/beam/2.14.0: ./ apache-beam-2.14.0-source-release.zip apache-beam-2.14.0-source-release.zip.asc apache-beam-2.14.0-source-release.zip.sha512
Author: anton Date: Wed Jul 24 22:29:06 2019 New Revision: 35011 Log: Beam 2.14.0 Release Added: dev/beam/2.14.0/ dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip (with props) dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.asc dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.sha512 Added: dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip == Binary file - no diff available. Propchange: dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip -- svn:mime-type = application/octet-stream Added: dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.asc == --- dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.asc (added) +++ dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.asc Wed Jul 24 22:29:06 2019 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCgAdFiEEieL/yufpnPbmgnz+9zSfIxD/sZMFAl042vkACgkQ9zSfIxD/ +sZOdMxAAqcVYw4gKNVc3Zif1TyAUaWDwKVvoxgfiXs2Dd8lFXjqz/jyFAaHVH7FU +0WeOf8vSUR4H64GHX4F7l2MOirN2P5cR8mQxgvM6peQDf/n48moNV4VQtL2/WPfr +iqURSnCEYFZcudICT0Fho2kmNeVIw1iiozIjrXtIsOtZPfPEQNOl1tod+HpL3DNo +5ZjCUmuD5Ug+7ru8QmrI25Zti0D9kl57HjJA2rJs493S5bNGvPgyOuKnNpmd5Jbx +MBu0lEigFnM8dmCbh0OL4jOd+t0IqNoEvdna06F/UUWGpHcBCt9bXK8FHGp4Em8a +52SYKcVmpAvOG+SSQSmpQkV8UCKpMEaHPrlvvFFqwKBoUm7C0Cp9/pQ7OJtAg4p7 +KMbP/VH+iSTtbNGtP46cF7+4ippZUFH5HuDcByk03QTI/VuDESr1Zw5XJ9Ldyxz0 +W9vATtbpYBh1ZyltmfKaBFgJQQY02/san2GaI6O3nxY3qyOg0sS/Je0XAOq1C/13 +npNBaihGmY0ANlI1eoengpRb976skgewM5Nu2jSWz3fJX1LsjfVAWYl6xv87Dvqh +CRsm8/P8F2XfrjdEEcQdI6li8OQ8rJKiy34SzTNke9iBYgBOCElnrL7vod89c8B7 +pmELcqZwm4dH8QVy6u2FzTGzC9Pps5UVN/j9I99PrC5bEou9XgM= +=hS2h +-END PGP SIGNATURE- Added: dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.sha512 == --- dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.sha512 (added) +++ dev/beam/2.14.0/apache-beam-2.14.0-source-release.zip.sha512 Wed Jul 24 22:29:06 2019 @@ -0,0 +1 @@ +a7e3469513572e6ee95b26e60a28e44bb624e01af91df43ca49a14154ecb00cbd23fabeaa6f5e0e89e7bb9927daf147ca0dc976f61b6f07df47dbf7ad2c985d7 apache-beam-2.14.0-source-release.zip
[beam] branch release-2.14.0 updated: Revert "Moving to 2.15.0-SNAPSHOT on master branch."
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new 9b49e4d Revert "Moving to 2.15.0-SNAPSHOT on master branch." new ae926f9 Merge pull request #9148 from akedin/revert-move-to-215-on-214-branch 9b49e4d is described below commit 9b49e4dab96b5e4b6abfccdd1e8ddba396f67ee6 Author: Anton Kedin AuthorDate: Wed Jul 24 13:47:51 2019 -0700 Revert "Moving to 2.15.0-SNAPSHOT on master branch." This reverts commit 6d218777a7bef0679128db2fec5ec257987f0890. Should only have been on master. --- buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy | 2 +- gradle.properties | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 5aa1bdc..7a62ae2 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -314,7 +314,7 @@ class BeamModulePlugin implements Plugin { // Automatically use the official release version if we are performing a release // otherwise append '-SNAPSHOT' -project.version = '2.15.0' +project.version = '2.14.0' if (!isRelease(project)) { project.version += '-SNAPSHOT' } diff --git a/gradle.properties b/gradle.properties index 299a616..f0ed4b4 100644 --- a/gradle.properties +++ b/gradle.properties @@ -23,4 +23,4 @@ offlineRepositoryRoot=offline-repository signing.gnupg.executable=gpg signing.gnupg.useLegacyGpg=true -version=2.15.0-SNAPSHOT +version=2.14.0-SNAPSHOT
[beam] branch release-2.14.0 updated: Relax pydot requirements.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new b9352da Relax pydot requirements. new 5340893 Merge pull request #9124 from tvalentyn/cp_9113 b9352da is described below commit b9352dac4b77ac8c5b98499f2dfab69ebdde21a4 Author: tvalentyn AuthorDate: Fri Jul 19 18:03:29 2019 -0700 Relax pydot requirements. --- sdks/python/setup.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/python/setup.py b/sdks/python/setup.py index 3106a21..8c7eb6c 100644 --- a/sdks/python/setup.py +++ b/sdks/python/setup.py @@ -120,7 +120,7 @@ REQUIRED_PACKAGES = [ # [BEAM-6287] pyarrow is not supported on Windows for Python 2 ('pyarrow>=0.11.1,<0.15.0; python_version >= "3.0" or ' 'platform_system != "Windows"'), -'pydot>=1.2.0,<1.3', +'pydot>=1.2.0,<2', 'pytz>=2018.3', # [BEAM-5628] Beam VCF IO is not supported in Python 3. 'pyvcf>=0.6.8,<0.7.0; python_version < "3.0"',
[beam] branch master updated: [BEAM-7783] Adding BeamTableStatistics.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 3a0d9c4 [BEAM-7783] Adding BeamTableStatistics. new ff7a803 Merge pull request #9104 from riazela/BeamTableStatistics 3a0d9c4 is described below commit 3a0d9c4fe16ced30a223557a4ad531365d4977ec Author: Alireza Samadian AuthorDate: Thu Jul 18 16:53:54 2019 -0700 [BEAM-7783] Adding BeamTableStatistics. --- .../apache/beam/sdk/io/TextRowCountEstimator.java | 6 +- .../beam/sdk/io/TextRowCountEstimatorTest.java | 12 +-- .../beam/sdk/extensions/sql/BeamSqlTable.java | 6 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 10 +-- .../sql/impl/BeamRowCountStatistics.java | 44 -- .../extensions/sql/impl/BeamTableStatistics.java | 93 ++ .../extensions/sql/impl/rel/BeamIOSourceRel.java | 24 -- .../sql/meta/provider/bigquery/BigQueryTable.java | 14 ++-- .../sql/meta/provider/test/TestTableProvider.java | 9 +-- .../sql/meta/provider/text/TextTable.java | 15 ++-- .../sql/impl/rule/JoinReorderingTest.java | 16 ++-- .../meta/provider/bigquery/BigQueryRowCountIT.java | 13 ++- .../meta/provider/bigquery/BigQueryTestTable.java | 4 +- 13 files changed, 160 insertions(+), 106 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/TextRowCountEstimator.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/TextRowCountEstimator.java index d220505..ad26fb1 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/TextRowCountEstimator.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/TextRowCountEstimator.java @@ -72,7 +72,7 @@ public abstract class TextRowCountEstimator { * @throws org.apache.beam.sdk.io.TextRowCountEstimator.NoEstimationException if all the sampled * lines are empty and we have not read all the lines in the matched files. */ - public Long estimateRowCount(PipelineOptions pipelineOptions) + public Double estimateRowCount(PipelineOptions pipelineOptions) throws IOException, NoEstimationException { long linesSize = 0; int numberOfReadLines = 0; @@ -129,7 +129,7 @@ public abstract class TextRowCountEstimator { } if (numberOfReadLines == 0 && sampledEverything) { - return 0L; + return 0d; } if (numberOfReadLines == 0) { @@ -138,7 +138,7 @@ public abstract class TextRowCountEstimator { } // This is total file sizes divided by average line size. -return totalFileSizes * numberOfReadLines / linesSize; +return (double) totalFileSizes * numberOfReadLines / linesSize; } /** Builder for {@link org.apache.beam.sdk.io.TextRowCountEstimator}. */ diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/io/TextRowCountEstimatorTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/io/TextRowCountEstimatorTest.java index b7e3f8e..6f53d1e 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/io/TextRowCountEstimatorTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/io/TextRowCountEstimatorTest.java @@ -59,16 +59,16 @@ public class TextRowCountEstimatorTest { writer.close(); TextRowCountEstimator textRowCountEstimator = TextRowCountEstimator.builder().setFilePattern(temporaryFolder.getRoot() + "/**").build(); -Long rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); +Double rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); Assert.assertNotNull(rows); -Assert.assertEquals(150L, rows.longValue()); +Assert.assertEquals(150d, rows, 0.01); } @Test(expected = FileNotFoundException.class) public void testEmptyFolder() throws Exception { TextRowCountEstimator textRowCountEstimator = TextRowCountEstimator.builder().setFilePattern(temporaryFolder.getRoot() + "/**").build(); -Long rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); +Double rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); } @Test @@ -82,8 +82,8 @@ public class TextRowCountEstimatorTest { writer.close(); TextRowCountEstimator textRowCountEstimator = TextRowCountEstimator.builder().setFilePattern(temporaryFolder.getRoot() + "/**").build(); -Long rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); -Assert.assertEquals(0L, rows.longValue()); +Double rows = textRowCountEstimator.estimateRowCount(PipelineOptionsFactory.create()); +Assert.assertEquals(0d, rows, 0.01); } @Test(expected = TextRowCountEstimator.NoEstimationException.class) @@ -110,7 +110,7 @@ public class TextRowCountEstimatorTest { TextRowCountEstimator.
[beam] branch master updated (76bc70d -> 8656d4c)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 76bc70d [BEAM-7484] Metrics collection in BigQuery perf tests (#8766) new a223b09 [BEAM-7545] Reordering Beam Joins and check if the produced join is correct. new 8656d4c Merge pull request #9040 from riazela/JoinReordering The 22312 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 15 +- .../extensions/sql/impl/CalciteQueryPlanner.java | 10 +- .../extensions/sql/impl/planner/BeamRuleSets.java | 7 + .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 33 +- .../sql/impl/rule/BeamJoinAssociateRule.java | 52 +++ .../sql/impl/rule/BeamJoinPushThroughJoinRule.java | 70 .../sql/impl/rule/JoinRelOptRuleCall.java | 105 + .../sql/meta/provider/test/TestTableProvider.java | 9 + .../sql/impl/rule/JoinReorderingTest.java | 461 + 9 files changed, 749 insertions(+), 13 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rule/BeamJoinAssociateRule.java create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rule/BeamJoinPushThroughJoinRule.java create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rule/JoinRelOptRuleCall.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rule/JoinReorderingTest.java
[beam] branch release-2.14.0 updated: [BEAM-7689] make a temporary directory unique for FileBaseSink
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new a7449df [BEAM-7689] make a temporary directory unique for FileBaseSink new 3187089 Merge pull request #9039 from ihji/cherry-pick-7689 a7449df is described below commit a7449df21c28db320f9757a44953f3b09d2002e3 Author: Heejong Lee AuthorDate: Mon Jul 8 14:18:42 2019 -0700 [BEAM-7689] make a temporary directory unique for FileBaseSink --- .../java/org/apache/beam/sdk/io/FileBasedSink.java | 20 1 file changed, 4 insertions(+), 16 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java index eff8a7c..2c21d5f 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java @@ -40,7 +40,6 @@ import java.util.Map; import java.util.Objects; import java.util.Set; import java.util.UUID; -import java.util.concurrent.atomic.AtomicLong; import javax.annotation.Nullable; import org.apache.beam.sdk.annotations.Experimental; import org.apache.beam.sdk.annotations.Experimental.Kind; @@ -82,9 +81,6 @@ import org.apache.beam.vendor.guava.v20_0.com.google.common.collect.Iterables; import org.apache.beam.vendor.guava.v20_0.com.google.common.collect.Lists; import org.apache.beam.vendor.guava.v20_0.com.google.common.collect.Maps; import org.apache.beam.vendor.guava.v20_0.com.google.common.collect.Sets; -import org.joda.time.Instant; -import org.joda.time.format.DateTimeFormat; -import org.joda.time.format.DateTimeFormatter; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -510,7 +506,7 @@ public abstract class FileBasedSink * * Default is a uniquely named subdirectory of the provided tempDirectory, e.g. if * tempDirectory is /path/to/foo/, the temporary directory will be - * /path/to/foo/temp-beam-foo-$date. + * /path/to/foo/.temp-beam-$uuid. * * @param sink the FileBasedSink that will be used to configure this write operation. */ @@ -522,20 +518,12 @@ public abstract class FileBasedSink private static class TemporaryDirectoryBuilder implements SerializableFunction { - private static final AtomicLong TEMP_COUNT = new AtomicLong(0); - private static final DateTimeFormatter TEMPDIR_TIMESTAMP = - DateTimeFormat.forPattern("-MM-dd_HH-mm-ss"); - // The intent of the code is to have a consistent value of tempDirectory across - // all workers, which wouldn't happen if now() was called inline. - private final String timestamp = Instant.now().toString(TEMPDIR_TIMESTAMP); - // Multiple different sinks may be used in the same output directory; use tempId to create a - // separate temp directory for each. - private final Long tempId = TEMP_COUNT.getAndIncrement(); + private final UUID tempUUID = UUID.randomUUID(); @Override public ResourceId apply(ResourceId tempDirectory) { -// Temp directory has a timestamp and a unique ID -String tempDirName = String.format(TEMP_DIRECTORY_PREFIX + "-%s-%s", timestamp, tempId); +// Temp directory has a random UUID postfix (BEAM-7689) +String tempDirName = String.format(TEMP_DIRECTORY_PREFIX + "-%s", tempUUID); return tempDirectory .getCurrentDirectory() .resolve(tempDirName, StandardResolveOptions.RESOLVE_DIRECTORY);
[beam] branch master updated: [BEAM-7729] Fixes the bug by checking the value first before parsing it.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 2215b77 [BEAM-7729] Fixes the bug by checking the value first before parsing it. new 60f70bf Merge pull request #9045 from riazela/BigQueryNullableBugFix 2215b77 is described below commit 2215b775cc10809ca330c916b45de7b914e56a83 Author: Alireza Samadian AuthorDate: Thu Jul 11 16:43:29 2019 -0700 [BEAM-7729] Fixes the bug by checking the value first before parsing it. --- .../java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java | 8 1 file changed, 8 insertions(+) diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java index 07102b3..6ff968b 100644 --- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java +++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java @@ -473,6 +473,14 @@ public class BigQueryUtils { public static Object convertAvroFormat( Field beamField, Object avroValue, BigQueryUtils.ConversionOptions options) { TypeName beamFieldTypeName = beamField.getType().getTypeName(); +if (avroValue == null) { + if (beamField.getType().getNullable()) { +return null; + } else { +throw new IllegalArgumentException( +String.format("Field %s not nullable", beamField.getName())); + } +} switch (beamFieldTypeName) { case INT16: case INT32:
[beam] branch release-2.14.0 updated: [BEAM-7424] Retry HTTP 429 errors from GCS
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new 5208659 [BEAM-7424] Retry HTTP 429 errors from GCS new 0bdb818 Merge pull request #9014 from ihji/BEAM-7424 5208659 is described below commit 5208659728734bd7fc991a86d72b80e816c2d915 Author: Heejong Lee AuthorDate: Mon Jun 24 11:55:52 2019 -0700 [BEAM-7424] Retry HTTP 429 errors from GCS match Python SDK retrying threshold to Java SDK threshold. --- sdks/python/apache_beam/io/gcp/gcsio.py| 3 ++- .../apache_beam/io/gcp/internal/clients/storage/storage_v1_client.py | 2 +- 2 files changed, 3 insertions(+), 2 deletions(-) diff --git a/sdks/python/apache_beam/io/gcp/gcsio.py b/sdks/python/apache_beam/io/gcp/gcsio.py index 5586eb4..6a27c27 100644 --- a/sdks/python/apache_beam/io/gcp/gcsio.py +++ b/sdks/python/apache_beam/io/gcp/gcsio.py @@ -522,7 +522,8 @@ class GcsDownloader(Downloader): # Initialize read buffer state. self._download_stream = io.BytesIO() self._downloader = transfer.Download( -self._download_stream, auto_transfer=False, chunksize=self._buffer_size) +self._download_stream, auto_transfer=False, chunksize=self._buffer_size, +num_retries=20) self._client.objects.Get(self._get_request, download=self._downloader) @retry.with_exponential_backoff( diff --git a/sdks/python/apache_beam/io/gcp/internal/clients/storage/storage_v1_client.py b/sdks/python/apache_beam/io/gcp/internal/clients/storage/storage_v1_client.py index cb5a0c9..735b159 100644 --- a/sdks/python/apache_beam/io/gcp/internal/clients/storage/storage_v1_client.py +++ b/sdks/python/apache_beam/io/gcp/internal/clients/storage/storage_v1_client.py @@ -52,7 +52,7 @@ class StorageV1(base_api.BaseApiClient): super(StorageV1, self).__init__( url, credentials=credentials, get_credentials=get_credentials, http=http, model=model, -log_request=log_request, log_response=log_response, +log_request=log_request, log_response=log_response, num_retries=20, credentials_args=credentials_args, default_global_params=default_global_params, additional_http_headers=additional_http_headers,
[beam] branch release-2.14.0 updated: BEAM-7548 fix flaky tests for ApproximateUnique
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new c576d59 BEAM-7548 fix flaky tests for ApproximateUnique new c1ce23f Merge pull request #8959 from Hannah-Jiang/release-2.14.0-check-pick-BEAM-7548 c576d59 is described below commit c576d59043fd3ad0c621c0d2f7b4864340a36449 Author: Hannah Jiang AuthorDate: Thu Jun 27 14:39:55 2019 -0700 BEAM-7548 fix flaky tests for ApproximateUnique --- sdks/python/apache_beam/transforms/stats_test.py | 81 +--- 1 file changed, 44 insertions(+), 37 deletions(-) diff --git a/sdks/python/apache_beam/transforms/stats_test.py b/sdks/python/apache_beam/transforms/stats_test.py index d8760a8..b86c72b 100644 --- a/sdks/python/apache_beam/transforms/stats_test.py +++ b/sdks/python/apache_beam/transforms/stats_test.py @@ -21,11 +21,11 @@ from __future__ import division import math import random -import sys import unittest from collections import defaultdict -import numpy as np +from tenacity import retry +from tenacity import stop_after_attempt import apache_beam as beam from apache_beam.testing.test_pipeline import TestPipeline @@ -34,7 +34,10 @@ from apache_beam.testing.util import equal_to class ApproximateUniqueTest(unittest.TestCase): - """Unit tests for ApproximateUnique.Globally and ApproximateUnique.PerKey.""" + """Unit tests for ApproximateUnique.Globally and ApproximateUnique.PerKey. + Hash() with Python3 is nondeterministic, so Approximation algorithm generates + different result each time and sometimes error rate is out of range, so add + retries for all tests who actually running approximation algorithm.""" def test_approximate_unique_global_by_invalid_size(self): # test if the transformation throws an error as expected with an invalid @@ -152,15 +155,16 @@ class ApproximateUniqueTest(unittest.TestCase): assert beam.ApproximateUnique._get_sample_size_from_est_error(0.05) == 1600 assert beam.ApproximateUnique._get_sample_size_from_est_error(0.01) == 4 - @unittest.skipIf(sys.version_info < (3, 0, 0), - 'Skip with py27 because hash function is not good enough.') + @unittest.skip('Skip it because hash function is not good enough. ' + 'TODO: BEAM-7654') def test_approximate_unique_global_by_sample_size(self): # test if estimation error with a given sample size is not greater than -# expected max error (sample size = 50% of population). -sample_size = 50 +# expected max error. +sample_size = 16 max_err = 2 / math.sqrt(sample_size) -random.seed(1) -test_input = [random.randint(0, 1000) for _ in range(100)] +test_input = [4, 34, 29, 46, 80, 66, 51, 81, 31, 9, 26, 36, 10, 41, 90, 35, + 33, 19, 88, 86, 28, 93, 38, 76, 15, 87, 12, 39, 84, 13, 32, + 49, 65, 100, 16, 27, 23, 30, 96, 54] actual_count = len(set(test_input)) pipeline = TestPipeline() @@ -176,6 +180,7 @@ class ApproximateUniqueTest(unittest.TestCase): label='assert:global_by_size') pipeline.run() + @retry(reraise=True, stop=stop_after_attempt(5)) def test_approximate_unique_global_by_sample_size_with_duplicates(self): # test if estimation error with a given sample size is not greater than # expected max error with duplicated input. @@ -197,11 +202,14 @@ class ApproximateUniqueTest(unittest.TestCase): label='assert:global_by_size_with_duplicates') pipeline.run() + @retry(reraise=True, stop=stop_after_attempt(5)) def test_approximate_unique_global_by_sample_size_with_small_population(self): # test if estimation is exactly same to actual value when sample size is # not smaller than population size (sample size > 100% of population). sample_size = 31 -test_input = [random.randint(0, 1000) for _ in range(30)] +test_input = [144, 160, 229, 923, 390, 756, 674, 769, 145, 888, + 809, 159, 222, 101, 943, 901, 876, 194, 232, 631, + 221, 829, 965, 729, 35, 33, 115, 894, 827, 364] actual_count = len(set(test_input)) pipeline = TestPipeline() @@ -214,13 +222,14 @@ class ApproximateUniqueTest(unittest.TestCase): label='assert:global_by_sample_size_with_small_population') pipeline.run() - @unittest.skipIf(sys.version_info < (3, 0, 0), - 'Skip with py27 because hash function is not good enough.') + @unittest.skip('Skip it because hash function is not good enough. ' + 'TODO: BEAM-7654') def test_approximate_unique_global_by_error(self): # test if estimation error from input error is not greater than input error. est_err = 0.3 -random.
[beam] branch release-2.14.0 updated: Merge pull request #8925 from apache/aaltay-patch-1
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new 98de63d Merge pull request #8925 from apache/aaltay-patch-1 new 5911b65 Merge pull request #8926 from aaltay/release-2.14.0 98de63d is described below commit 98de63d77d358c53fb025982a6ed773d1d99bcb3 Author: Ahmet Altay AuthorDate: Fri Jun 21 11:46:26 2019 -0700 Merge pull request #8925 from apache/aaltay-patch-1 [BEAM-7616] Add a timeout to urlopen calls --- sdks/python/apache_beam/internal/gcp/auth.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/python/apache_beam/internal/gcp/auth.py b/sdks/python/apache_beam/internal/gcp/auth.py index 1c6d8e2..8b331e8 100644 --- a/sdks/python/apache_beam/internal/gcp/auth.py +++ b/sdks/python/apache_beam/internal/gcp/auth.py @@ -93,7 +93,7 @@ class _GCEMetadataCredentials(OAuth2Credentials): token_url = ('http://{}/computeMetadata/v1/instance/service-accounts/' 'default/token').format(metadata_root) req = Request(token_url, headers={'Metadata-Flavor': 'Google'}) -token_data = json.loads(urlopen(req).read().decode('utf-8')) +token_data = json.loads(urlopen(req, timeout=60).read().decode('utf-8')) self.access_token = token_data['access_token'] self.token_expiry = (refresh_time + datetime.timedelta(seconds=token_data['expires_in']))
[beam] branch release-2.14.0 updated: Update pyarrow version requirement in setup.py
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new 8b5b8a2 Update pyarrow version requirement in setup.py new ae5d679 Merge pull request #9030 from paulgc/release-2.14.0 8b5b8a2 is described below commit 8b5b8a2683a9201fa8780852eb148f5cef5099ca Author: Paul Suganthan AuthorDate: Tue Jul 9 13:05:26 2019 -0700 Update pyarrow version requirement in setup.py --- sdks/python/setup.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/python/setup.py b/sdks/python/setup.py index 24ada13..3106a21 100644 --- a/sdks/python/setup.py +++ b/sdks/python/setup.py @@ -118,7 +118,7 @@ REQUIRED_PACKAGES = [ # grpcio 1.8.1 and above requires protobuf 3.5.0.post1. 'protobuf>=3.5.0.post1,<4', # [BEAM-6287] pyarrow is not supported on Windows for Python 2 -('pyarrow>=0.11.1,<0.14.0; python_version >= "3.0" or ' +('pyarrow>=0.11.1,<0.15.0; python_version >= "3.0" or ' 'platform_system != "Windows"'), 'pydot>=1.2.0,<1.3', 'pytz>=2018.3',
[beam] branch release-2.14.0 updated (524272b -> 18f64aa)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git. from 524272b Cherry pick #8954: Spotless config update to include java files only under src directory add 18f64aa [BEAM-7603] Support for ValueProvider-given GCS Location for WriteToBigQuery w File Loads (#8908) No new revisions were added by this update. Summary of changes: .../apache_beam/io/gcp/bigquery_file_loads.py | 47 ++ 1 file changed, 30 insertions(+), 17 deletions(-)
[beam] branch master updated: [BEAM-7545] Adding RowCount to TextTable.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 16ceca5 [BEAM-7545] Adding RowCount to TextTable. new 3d576f7 Merge pull request #8951 from riazela/TextTableRowCount 16ceca5 is described below commit 16ceca59d8cb823f0f5e42cd7dc6ea717bef3ec8 Author: Alireza Samadian AuthorDate: Mon Jun 24 16:26:29 2019 -0700 [BEAM-7545] Adding RowCount to TextTable. --- .../main/java/org/apache/beam/sdk/io/FileIO.java | 82 +--- .../apache/beam/sdk/io/TextRowCountEstimator.java | 219 + .../beam/sdk/io/TextRowCountEstimatorTest.java | 116 +++ .../sql/meta/provider/text/TextTable.java | 36 4 files changed, 424 insertions(+), 29 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileIO.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileIO.java index 5447e86..57a89ac 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileIO.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileIO.java @@ -729,6 +729,55 @@ public class FileIO { builder.add(DisplayData.item("directoryTreatment", getDirectoryTreatment().toString())); } +/** + * @return True if metadata is a directory and directory Treatment is SKIP. + * @throws java.lang.IllegalArgumentException if metadata is a directory and directoryTreatment + * is Prohibited. + * @throws java.lang.UnsupportedOperationException if metadata is a directory and + * directoryTreatment is not SKIP or PROHIBIT. + */ +static boolean shouldSkipDirectory( +MatchResult.Metadata metadata, DirectoryTreatment directoryTreatment) { + if (metadata.resourceId().isDirectory()) { +switch (directoryTreatment) { + case SKIP: +return true; + case PROHIBIT: +throw new IllegalArgumentException( +"Trying to read " + metadata.resourceId() + " which is a directory"); + + default: +throw new UnsupportedOperationException( +"Unknown DirectoryTreatment: " + directoryTreatment); +} + } + + return false; +} + +/** + * Converts metadata to readableFile. Make sure {@link + * #shouldSkipDirectory(org.apache.beam.sdk.io.fs.MatchResult.Metadata, + * org.apache.beam.sdk.io.FileIO.ReadMatches.DirectoryTreatment)} returns false before using. + */ +static ReadableFile matchToReadableFile( +MatchResult.Metadata metadata, Compression compression) { + + compression = + (compression == Compression.AUTO) + ? Compression.detect(metadata.resourceId().getFilename()) + : compression; + return new ReadableFile( + MatchResult.Metadata.builder() + .setResourceId(metadata.resourceId()) + .setSizeBytes(metadata.sizeBytes()) + .setLastModifiedMillis(metadata.lastModifiedMillis()) + .setIsReadSeekEfficient( + metadata.isReadSeekEfficient() && compression == Compression.UNCOMPRESSED) + .build(), + compression); +} + private static class ToReadableFileFn extends DoFn { private final ReadMatches spec; @@ -738,36 +787,11 @@ public class FileIO { @ProcessElement public void process(ProcessContext c) { -MatchResult.Metadata metadata = c.element(); -if (metadata.resourceId().isDirectory()) { - switch (spec.getDirectoryTreatment()) { -case SKIP: - return; - -case PROHIBIT: - throw new IllegalArgumentException( - "Trying to read " + metadata.resourceId() + " which is a directory"); - -default: - throw new UnsupportedOperationException( - "Unknown DirectoryTreatment: " + spec.getDirectoryTreatment()); - } +if (shouldSkipDirectory(c.element(), spec.getDirectoryTreatment())) { + return; } - -Compression compression = -(spec.getCompression() == Compression.AUTO) -? Compression.detect(metadata.resourceId().getFilename()) -: spec.getCompression(); -c.output( -new ReadableFile( -MatchResult.Metadata.builder() -.setResourceId(metadata.resourceId()) -.setSizeBytes(metadata.sizeBytes()) -.setLastModifiedMillis(metadata.lastModifiedMillis()) -.setIsReadSeekEfficient( -metadata.isReadSeekEfficient() && compression == Compression.UNCOMPRESSED) -.build(), -
[beam] branch release-2.14.0 updated: Match Python 3 warning message in __init__.py with the one in setup.py.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.14.0 by this push: new 38672fd Match Python 3 warning message in __init__.py with the one in setup.py. new 09ea2ca Merge pull request #8958 from tvalentyn/py3_warning_cp 38672fd is described below commit 38672fdffef6ece2d679bf45be1cc0a63088f93b Author: Valentyn Tymofieiev AuthorDate: Thu Jun 27 10:23:49 2019 -0700 Match Python 3 warning message in __init__.py with the one in setup.py. --- sdks/python/apache_beam/__init__.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/sdks/python/apache_beam/__init__.py b/sdks/python/apache_beam/__init__.py index e06cc69..4a0701b 100644 --- a/sdks/python/apache_beam/__init__.py +++ b/sdks/python/apache_beam/__init__.py @@ -81,8 +81,8 @@ import warnings if sys.version_info[0] == 3: warnings.warn( - 'Running the Apache Beam SDK on Python 3 is not yet fully supported. ' - 'You may encounter buggy behavior or missing features.') + 'Some syntactic constructs of Python 3 are not yet fully supported by ' + 'Apache Beam.') elif sys.version_info[0] == 2 and sys.version_info[1] == 7: pass else:
[beam] branch master updated: Spotless config update to include java files only under src directory
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e6cc73b Spotless config update to include java files only under src directory new c39ba4a Merge pull request #8954 from akedin/spotless-include-src e6cc73b is described below commit e6cc73b0ffc20b0225ad712ddd2d42c4ca257eb1 Author: akedin AuthorDate: Wed Jun 26 16:20:14 2019 -0700 Spotless config update to include java files only under src directory --- .../main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy | 9 ++--- 1 file changed, 2 insertions(+), 7 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 725880f..aa16673 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -816,13 +816,8 @@ class BeamModulePlugin implements Plugin { project.sourceSets.each { sourceSet -> targetFiles += sourceSet.allJava } - target targetFiles.matching { -include '**/*.java' -exclude '**/archetype-resources/src/**' -exclude '**/build/generated/**' -exclude '**/build/generated-src/**' -exclude '**/build/generated-*-avro-*/**' - } + + target targetFiles.matching { include 'src/*/java/**/*.java' } } }
[beam] branch master updated: Match Python 3 warning message in __init__.py with the one in setup.py.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new c44f352 Match Python 3 warning message in __init__.py with the one in setup.py. new 063091e Merge pull request #8956 from tvalentyn/classifiers c44f352 is described below commit c44f35252d7c1f07a143564be4ea8717adb9 Author: Valentyn Tymofieiev AuthorDate: Thu Jun 27 10:23:49 2019 -0700 Match Python 3 warning message in __init__.py with the one in setup.py. --- sdks/python/apache_beam/__init__.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/sdks/python/apache_beam/__init__.py b/sdks/python/apache_beam/__init__.py index e06cc69..4a0701b 100644 --- a/sdks/python/apache_beam/__init__.py +++ b/sdks/python/apache_beam/__init__.py @@ -81,8 +81,8 @@ import warnings if sys.version_info[0] == 3: warnings.warn( - 'Running the Apache Beam SDK on Python 3 is not yet fully supported. ' - 'You may encounter buggy behavior or missing features.') + 'Some syntactic constructs of Python 3 are not yet fully supported by ' + 'Apache Beam.') elif sys.version_info[0] == 2 and sys.version_info[1] == 7: pass else:
[beam] 01/01: Create release branch for version 2.14.0.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git commit b4704ee530c7f769e1f306c2dec3f0cbe6c5bfd1 Author: Anton Kedin AuthorDate: Wed Jun 19 16:02:10 2019 -0700 Create release branch for version 2.14.0. --- runners/google-cloud-dataflow-java/build.gradle | 2 +- sdks/python/apache_beam/version.py | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) diff --git a/runners/google-cloud-dataflow-java/build.gradle b/runners/google-cloud-dataflow-java/build.gradle index 35ea595..8e0798a 100644 --- a/runners/google-cloud-dataflow-java/build.gradle +++ b/runners/google-cloud-dataflow-java/build.gradle @@ -39,7 +39,7 @@ processResources { filter org.apache.tools.ant.filters.ReplaceTokens, tokens: [ 'dataflow.legacy_environment_major_version' : '7', 'dataflow.fnapi_environment_major_version' : '7', -'dataflow.container_version' : 'beam-master-20190415' +'dataflow.container_version' : 'beam-2.14.0' ] } diff --git a/sdks/python/apache_beam/version.py b/sdks/python/apache_beam/version.py index 1688685..c9fa3b8 100644 --- a/sdks/python/apache_beam/version.py +++ b/sdks/python/apache_beam/version.py @@ -18,4 +18,4 @@ """Apache Beam SDK version information and utilities.""" -__version__ = '2.15.0.dev' +__version__ = '2.14.0'
[beam] branch release-2.14.0 created (now b4704ee)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch release-2.14.0 in repository https://gitbox.apache.org/repos/asf/beam.git. at b4704ee Create release branch for version 2.14.0. This branch includes the following new commits: new b4704ee Create release branch for version 2.14.0. The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference.
[beam] branch master updated: [BEAM-7513] Adding RowCount to BigQueryTable.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e868b80 [BEAM-7513] Adding RowCount to BigQueryTable. new b2fd4e3 Merge pull request #8892 from riazela/bigquery-rowcount e868b80 is described below commit e868b8043287cece938a1864066f33ec33f813e8 Author: Alireza Samadian AuthorDate: Wed Jun 19 09:36:19 2019 -0700 [BEAM-7513] Adding RowCount to BigQueryTable. --- .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 7 +- .../beam/sdk/extensions/sql/BeamSqlTable.java | 7 + .../beam/sdk/extensions/sql/SqlTransform.java | 2 + .../sdk/extensions/sql/impl/BeamCalciteSchema.java | 5 +- .../sql/impl/BeamCalciteSchemaFactory.java | 12 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 49 ++- .../BeamRowCountStatistics.java} | 37 +++-- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 26 +++- .../sdk/extensions/sql/impl/JdbcConnection.java| 9 ++ .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 6 +- .../sql/meta/provider/bigquery/BigQueryTable.java | 39 + .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 7 +- .../sdk/extensions/sql/impl/JdbcDriverTest.java| 31 ++-- .../extensions/sql/impl/parser/BeamDDLTest.java| 7 +- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 +- .../meta/provider/bigquery/BigQueryRowCountIT.java | 161 + .../meta/provider/bigquery/BigQueryTestTable.java | 45 ++ .../bigquery/BigQueryTestTableProvider.java| 71 + .../sql/meta/provider/pubsub/PubsubJsonIT.java | 2 +- .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 19 +++ 20 files changed, 498 insertions(+), 51 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java index 5e44c6c..8bdb1bf 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java @@ -25,6 +25,7 @@ import org.apache.beam.sdk.extensions.sql.impl.rel.BeamEnumerableConverter; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamSqlRelUtils; import org.apache.beam.sdk.extensions.sql.meta.store.MetaStore; import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.options.PipelineOptionsFactory; /** {@link BeamSqlCli} provides methods to execute Beam SQL with an interactive client. */ @Experimental @@ -34,15 +35,17 @@ public class BeamSqlCli { private MetaStore metaStore; public BeamSqlCli metaStore(MetaStore metaStore) { -return metaStore(metaStore, false); +return metaStore(metaStore, false, PipelineOptionsFactory.create()); } - public BeamSqlCli metaStore(MetaStore metaStore, boolean autoLoadUdfUdaf) { + public BeamSqlCli metaStore( + MetaStore metaStore, boolean autoLoadUdfUdaf, PipelineOptions pipelineOptions) { this.metaStore = metaStore; BeamSqlEnv.BeamSqlEnvBuilder builder = BeamSqlEnv.builder(metaStore); if (autoLoadUdfUdaf) { builder.autoLoadUserDefinedFunctions(); } +builder.setPipelineOptions(pipelineOptions); this.env = builder.build(); return this; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java index 14f1b80..63f7158 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java @@ -17,6 +17,8 @@ */ package org.apache.beam.sdk.extensions.sql; +import org.apache.beam.sdk.extensions.sql.impl.BeamRowCountStatistics; +import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.schemas.Schema; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; @@ -36,4 +38,9 @@ public interface BeamSqlTable { /** Get the schema info of the table. */ Schema getSchema(); + + /** Estimates the number of rows or returns null if there is no estimation. */ + default BeamRowCountStatistics getRowCount(PipelineOptions options) { +return BeamRowCountStatistics.UNKNOWN; + } } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java index e45daca..afa4438 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache
[beam] branch master updated: Moving to 2.15.0-SNAPSHOT on master branch.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 6d21877 Moving to 2.15.0-SNAPSHOT on master branch. 6d21877 is described below commit 6d218777a7bef0679128db2fec5ec257987f0890 Author: Anton Kedin AuthorDate: Wed Jun 19 15:27:20 2019 -0700 Moving to 2.15.0-SNAPSHOT on master branch. --- buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy | 2 +- gradle.properties | 2 +- sdks/python/apache_beam/version.py | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 699344f..66f8654 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -314,7 +314,7 @@ class BeamModulePlugin implements Plugin { // Automatically use the official release version if we are performing a release // otherwise append '-SNAPSHOT' -project.version = '2.14.0' +project.version = '2.15.0' if (!isRelease(project)) { project.version += '-SNAPSHOT' } diff --git a/gradle.properties b/gradle.properties index f0ed4b4..299a616 100644 --- a/gradle.properties +++ b/gradle.properties @@ -23,4 +23,4 @@ offlineRepositoryRoot=offline-repository signing.gnupg.executable=gpg signing.gnupg.useLegacyGpg=true -version=2.14.0-SNAPSHOT +version=2.15.0-SNAPSHOT diff --git a/sdks/python/apache_beam/version.py b/sdks/python/apache_beam/version.py index d4053e6..1688685 100644 --- a/sdks/python/apache_beam/version.py +++ b/sdks/python/apache_beam/version.py @@ -18,4 +18,4 @@ """Apache Beam SDK version information and utilities.""" -__version__ = '2.14.0.dev' +__version__ = '2.15.0.dev'
[beam] branch revert-8822-bigquery-rowcount deleted (was 7b7c215)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch revert-8822-bigquery-rowcount in repository https://gitbox.apache.org/repos/asf/beam.git. was 7b7c215 Revert "[BEAM-7513] Adding Row Count for Bigquery Table" The revisions that were on this branch are still contained in other references; therefore, this change does not discard any commits from the repository.
[beam] 01/01: Merge pull request #8882 from apache/revert-8822-bigquery-rowcount
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 35f99ba92ddc5728e767e14a3e7d7df0a8bc4bb5 Merge: 3c61ab8 7b7c215 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Mon Jun 17 20:50:31 2019 -0700 Merge pull request #8882 from apache/revert-8822-bigquery-rowcount Revert "[BEAM-7513] Adding Row Count for Bigquery Table" .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 7 +- .../beam/sdk/extensions/sql/BeamSqlTable.java | 7 - .../beam/sdk/extensions/sql/SqlTransform.java | 2 - .../sql/impl/BeamCalciteSchemaFactory.java | 12 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 23 --- .../sql/impl/BeamRowCountStatistics.java | 44 -- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 26 +--- .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 34 + .../sql/meta/provider/bigquery/BigQueryTable.java | 39 - .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 7 +- .../sdk/extensions/sql/impl/JdbcDriverTest.java| 31 ++-- .../extensions/sql/impl/parser/BeamDDLTest.java| 7 +- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 +- .../meta/provider/bigquery/BigQueryRowCountIT.java | 161 - .../meta/provider/bigquery/BigQueryTestTable.java | 45 -- .../bigquery/BigQueryTestTableProvider.java| 71 - .../sql/meta/provider/pubsub/PubsubJsonIT.java | 2 +- .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 23 --- 18 files changed, 29 insertions(+), 519 deletions(-)
[beam] branch master updated (3c61ab8 -> 35f99ba)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 3c61ab8 Merge pull request #8867 from tvalentyn/classifiers add 7b7c215 Revert "[BEAM-7513] Adding Row Count for Bigquery Table" new 35f99ba Merge pull request #8882 from apache/revert-8822-bigquery-rowcount The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 7 +- .../beam/sdk/extensions/sql/BeamSqlTable.java | 7 - .../beam/sdk/extensions/sql/SqlTransform.java | 2 - .../sql/impl/BeamCalciteSchemaFactory.java | 12 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 23 --- .../sql/impl/BeamRowCountStatistics.java | 44 -- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 26 +--- .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 34 + .../sql/meta/provider/bigquery/BigQueryTable.java | 39 - .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 7 +- .../sdk/extensions/sql/impl/JdbcDriverTest.java| 31 ++-- .../extensions/sql/impl/parser/BeamDDLTest.java| 7 +- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 +- .../meta/provider/bigquery/BigQueryRowCountIT.java | 161 - .../meta/provider/bigquery/BigQueryTestTable.java | 45 -- .../bigquery/BigQueryTestTableProvider.java| 71 - .../sql/meta/provider/pubsub/PubsubJsonIT.java | 2 +- .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 23 --- 18 files changed, 29 insertions(+), 519 deletions(-) delete mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamRowCountStatistics.java delete mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryRowCountIT.java delete mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTestTable.java delete mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTestTableProvider.java
[beam] branch revert-8822-bigquery-rowcount created (now 7b7c215)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch revert-8822-bigquery-rowcount in repository https://gitbox.apache.org/repos/asf/beam.git. at 7b7c215 Revert "[BEAM-7513] Adding Row Count for Bigquery Table" This branch includes the following new commits: new 7b7c215 Revert "[BEAM-7513] Adding Row Count for Bigquery Table" The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference.
[beam] 01/01: Revert "[BEAM-7513] Adding Row Count for Bigquery Table"
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch revert-8822-bigquery-rowcount in repository https://gitbox.apache.org/repos/asf/beam.git commit 7b7c21544b50b1c8ac96f6c88362a0e266b6d55e Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Mon Jun 17 16:48:30 2019 -0700 Revert "[BEAM-7513] Adding Row Count for Bigquery Table" --- .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 7 +- .../beam/sdk/extensions/sql/BeamSqlTable.java | 7 - .../beam/sdk/extensions/sql/SqlTransform.java | 2 - .../sql/impl/BeamCalciteSchemaFactory.java | 12 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 23 --- .../sql/impl/BeamRowCountStatistics.java | 44 -- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 26 +--- .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 34 + .../sql/meta/provider/bigquery/BigQueryTable.java | 39 - .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 7 +- .../sdk/extensions/sql/impl/JdbcDriverTest.java| 31 ++-- .../extensions/sql/impl/parser/BeamDDLTest.java| 7 +- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 +- .../meta/provider/bigquery/BigQueryRowCountIT.java | 161 - .../meta/provider/bigquery/BigQueryTestTable.java | 45 -- .../bigquery/BigQueryTestTableProvider.java| 71 - .../sql/meta/provider/pubsub/PubsubJsonIT.java | 2 +- .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 23 --- 18 files changed, 29 insertions(+), 519 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java index 8bdb1bf..5e44c6c 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java @@ -25,7 +25,6 @@ import org.apache.beam.sdk.extensions.sql.impl.rel.BeamEnumerableConverter; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamSqlRelUtils; import org.apache.beam.sdk.extensions.sql.meta.store.MetaStore; import org.apache.beam.sdk.options.PipelineOptions; -import org.apache.beam.sdk.options.PipelineOptionsFactory; /** {@link BeamSqlCli} provides methods to execute Beam SQL with an interactive client. */ @Experimental @@ -35,17 +34,15 @@ public class BeamSqlCli { private MetaStore metaStore; public BeamSqlCli metaStore(MetaStore metaStore) { -return metaStore(metaStore, false, PipelineOptionsFactory.create()); +return metaStore(metaStore, false); } - public BeamSqlCli metaStore( - MetaStore metaStore, boolean autoLoadUdfUdaf, PipelineOptions pipelineOptions) { + public BeamSqlCli metaStore(MetaStore metaStore, boolean autoLoadUdfUdaf) { this.metaStore = metaStore; BeamSqlEnv.BeamSqlEnvBuilder builder = BeamSqlEnv.builder(metaStore); if (autoLoadUdfUdaf) { builder.autoLoadUserDefinedFunctions(); } -builder.setPipelineOptions(pipelineOptions); this.env = builder.build(); return this; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java index 63f7158..14f1b80 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java @@ -17,8 +17,6 @@ */ package org.apache.beam.sdk.extensions.sql; -import org.apache.beam.sdk.extensions.sql.impl.BeamRowCountStatistics; -import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.schemas.Schema; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; @@ -38,9 +36,4 @@ public interface BeamSqlTable { /** Get the schema info of the table. */ Schema getSchema(); - - /** Estimates the number of rows or returns null if there is no estimation. */ - default BeamRowCountStatistics getRowCount(PipelineOptions options) { -return BeamRowCountStatistics.UNKNOWN; - } } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java index afa4438..e45daca 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java @@ -118,8 +118,6 @@ public abstract class SqlTransform extends PTransform> sqlEnvBuilder.setQueryPlannerClassName( input.getPipeline().getOptions().as(BeamSqlPipelineOptions.cl
[beam] branch master updated: [BEAM-7513] Implements row estimation for BigQuery.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new bc2c286 [BEAM-7513] Implements row estimation for BigQuery. new 2de7d07 Merge pull request #8822 from riazela/bigquery-rowcount bc2c286 is described below commit bc2c2863e400f28a4ac14ffb07b03cfe8a312a23 Author: Alireza Samadian AuthorDate: Tue Jun 11 09:09:23 2019 -0700 [BEAM-7513] Implements row estimation for BigQuery. --- .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 7 +- .../beam/sdk/extensions/sql/BeamSqlTable.java | 7 + .../beam/sdk/extensions/sql/SqlTransform.java | 2 + .../sql/impl/BeamCalciteSchemaFactory.java | 12 +- .../sdk/extensions/sql/impl/BeamCalciteTable.java | 23 +++ .../BeamRowCountStatistics.java} | 37 +++-- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 26 +++- .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 34 - .../sql/meta/provider/bigquery/BigQueryTable.java | 39 + .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 7 +- .../sdk/extensions/sql/impl/JdbcDriverTest.java| 31 ++-- .../extensions/sql/impl/parser/BeamDDLTest.java| 7 +- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 +- .../meta/provider/bigquery/BigQueryRowCountIT.java | 161 + .../meta/provider/bigquery/BigQueryTestTable.java | 45 ++ .../bigquery/BigQueryTestTableProvider.java| 71 + .../sql/meta/provider/pubsub/PubsubJsonIT.java | 2 +- .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 23 +++ 18 files changed, 496 insertions(+), 45 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java index 5e44c6c..8bdb1bf 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java @@ -25,6 +25,7 @@ import org.apache.beam.sdk.extensions.sql.impl.rel.BeamEnumerableConverter; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamSqlRelUtils; import org.apache.beam.sdk.extensions.sql.meta.store.MetaStore; import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.options.PipelineOptionsFactory; /** {@link BeamSqlCli} provides methods to execute Beam SQL with an interactive client. */ @Experimental @@ -34,15 +35,17 @@ public class BeamSqlCli { private MetaStore metaStore; public BeamSqlCli metaStore(MetaStore metaStore) { -return metaStore(metaStore, false); +return metaStore(metaStore, false, PipelineOptionsFactory.create()); } - public BeamSqlCli metaStore(MetaStore metaStore, boolean autoLoadUdfUdaf) { + public BeamSqlCli metaStore( + MetaStore metaStore, boolean autoLoadUdfUdaf, PipelineOptions pipelineOptions) { this.metaStore = metaStore; BeamSqlEnv.BeamSqlEnvBuilder builder = BeamSqlEnv.builder(metaStore); if (autoLoadUdfUdaf) { builder.autoLoadUserDefinedFunctions(); } +builder.setPipelineOptions(pipelineOptions); this.env = builder.build(); return this; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java index 14f1b80..63f7158 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlTable.java @@ -17,6 +17,8 @@ */ package org.apache.beam.sdk.extensions.sql; +import org.apache.beam.sdk.extensions.sql.impl.BeamRowCountStatistics; +import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.schemas.Schema; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; @@ -36,4 +38,9 @@ public interface BeamSqlTable { /** Get the schema info of the table. */ Schema getSchema(); + + /** Estimates the number of rows or returns null if there is no estimation. */ + default BeamRowCountStatistics getRowCount(PipelineOptions options) { +return BeamRowCountStatistics.UNKNOWN; + } } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java index e45daca..afa4438 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java @@ -118,6 +118,8 @@ public abstract class SqlTransform extends
[beam] branch master updated: [BEAM-7526] Fix toBeamValue logic in BigQueryUtils
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 2e17f12 [BEAM-7526] Fix toBeamValue logic in BigQueryUtils new 01f14a0 Merge pull request #8814 from charithe/beam-7526 2e17f12 is described below commit 2e17f12fa8fea43fb93a47f1770e163bac65cb3a Author: Charith Ellawala AuthorDate: Tue Jun 11 11:09:11 2019 +0100 [BEAM-7526] Fix toBeamValue logic in BigQueryUtils --- .../java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java | 7 +-- .../org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtilsTest.java | 9 +++-- 2 files changed, 12 insertions(+), 4 deletions(-) diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java index 1a87875..3ef1507 100644 --- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java +++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java @@ -444,12 +444,15 @@ public class BigQueryUtils { if (jsonBQValue instanceof List) { return ((List) jsonBQValue) .stream() + .map(v -> ((Map) v).get("v")) .map(v -> toBeamValue(fieldType.getCollectionElementType(), v)) .collect(toList()); } -if (jsonBQValue instanceof TableRow) { - return toBeamRow(fieldType.getRowSchema(), (TableRow) jsonBQValue); +if (jsonBQValue instanceof Map) { + TableRow tr = new TableRow(); + tr.putAll((Map) jsonBQValue); + return toBeamRow(fieldType.getRowSchema(), tr); } throw new UnsupportedOperationException( diff --git a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtilsTest.java b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtilsTest.java index 3315598..2226be9 100644 --- a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtilsTest.java +++ b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtilsTest.java @@ -131,7 +131,11 @@ public class BigQueryUtilsTest { Row.withSchema(ARRAY_TYPE).addValues((Object) Arrays.asList(123L, 124L)).build(); private static final TableRow BQ_ARRAY_ROW = - new TableRow().set("ids", Arrays.asList("123", "124")); + new TableRow() + .set( + "ids", + Arrays.asList( + Collections.singletonMap("v", "123"), Collections.singletonMap("v", "124"))); private static final Row ROW_ROW = Row.withSchema(ROW_TYPE).addValues(FLAT_ROW).build(); @@ -141,7 +145,8 @@ public class BigQueryUtilsTest { Row.withSchema(ARRAY_ROW_TYPE).addValues((Object) Arrays.asList(FLAT_ROW)).build(); private static final TableRow BQ_ARRAY_ROW_ROW = - new TableRow().set("rows", Collections.singletonList(BQ_FLAT_ROW)); + new TableRow() + .set("rows", Collections.singletonList(Collections.singletonMap("v", BQ_FLAT_ROW))); private static final TableSchema BQ_FLAT_TYPE = new TableSchema().setFields(Arrays.asList(ID, VALUE, NAME, TIMESTAMP, VALID));
[beam] branch master updated (f16b689 -> b7b0868)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from f16b689 Merge pull request #8810: [BEAM-7524] Update Python dependencies page for 2.13.0 new 3e0b352 [BEAM-7511] Fixes the bug in KafkaTable Initialization. new 550c1cc [BEAM-7511] Fixes the bug in KafkaTable Initialization, returned the deleted methods back. new 93d9334 [BEAM-7511] Fixes the bug in KafkaTable Initialization, returned the deleted methods back. new b7b0868 Merge pull request #8797 from riazela/master The 21882 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/meta/provider/kafka/BeamKafkaTable.java | 2 ++ 1 file changed, 2 insertions(+)
[beam] branch master updated: [SQL] Add array constructor test
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new b74cb17 [SQL] Add array constructor test new e8666a4 Merge pull request #8727 from akedin/array-constructor b74cb17 is described below commit b74cb17b4d3893611eb853775ff6d2528f8a1adf Author: akedin AuthorDate: Thu May 30 10:22:20 2019 -0700 [SQL] Add array constructor test --- .../apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java | 13 + 1 file changed, 13 insertions(+) diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java index fbdf625..1e6eaf3 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamComplexTypeTest.java @@ -165,6 +165,19 @@ public class BeamComplexTypeTest { } @Test + public void testArrayConstructor() { +BeamSqlEnv sqlEnv = BeamSqlEnv.inMemory(readOnlyTableProvider); +PCollection stream = +BeamSqlRelUtils.toPCollection(pipeline, sqlEnv.parseQuery("SELECT ARRAY[1, 2, 3] f_arr")); +PAssert.that(stream) +.containsInAnyOrder( +Row.withSchema(Schema.builder().addArrayField("f_arr", FieldType.INT32).build()) +.addValue(Arrays.asList(1, 2, 3)) +.build()); +pipeline.run().waitUntilFinish(Duration.standardMinutes(2)); + } + + @Test public void testRowWithArray() { BeamSqlEnv sqlEnv = BeamSqlEnv.inMemory(readOnlyTableProvider); PCollection stream =
[beam] branch master updated: Fix build break caused by #7776
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 466a17d Fix build break caused by #7776 new b715c45 Merge pull request #8726 from apilloud/break 466a17d is described below commit 466a17ddb1464b419859dc3ade3628fec591abf1 Author: Andrew Pilloud AuthorDate: Thu May 30 09:51:23 2019 -0700 Fix build break caused by #7776 --- .../org/apache/beam/sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java index 4895004..6f24bb2 100644 --- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java +++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java @@ -32,12 +32,12 @@ import javax.annotation.Nullable; import org.apache.beam.sdk.coders.CannotProvideCoderException; import org.apache.beam.sdk.coders.Coder; import org.apache.beam.sdk.coders.CoderRegistry; +import org.apache.beam.sdk.extensions.gcp.util.BackOffAdapter; import org.apache.beam.sdk.io.gcp.bigquery.BigQueryHelpers.JsonTableRefToTableSpec; import org.apache.beam.sdk.options.ValueProvider; import org.apache.beam.sdk.options.ValueProvider.NestedValueProvider; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.SerializableFunction; -import org.apache.beam.sdk.util.BackOffAdapter; import org.apache.beam.sdk.util.FluentBackoff; import org.apache.beam.sdk.values.PCollectionView; import org.apache.beam.sdk.values.ValueInSingleWindow;
[beam] branch master updated: [SQL] Add support for `FLOAT64` to DataCatalogTableProvider
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 5f772bd [SQL] Add support for `FLOAT64` to DataCatalogTableProvider new 5f83637 Merge pull request #8701 from akedin/dc-float64 5f772bd is described below commit 5f772bd77b37707df6dcc30ef15fb8f2410dd0b8 Author: akedin AuthorDate: Tue May 28 11:26:17 2019 -0700 [SQL] Add support for `FLOAT64` to DataCatalogTableProvider --- .../beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java | 1 + 1 file changed, 1 insertion(+) diff --git a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java index 69288a8..91ff5e9 100644 --- a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java +++ b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java @@ -39,6 +39,7 @@ class SchemaUtils { .put("DATETIME", FieldType.DATETIME) .put("DOUBLE", FieldType.DOUBLE) .put("FLOAT", FieldType.DOUBLE) + .put("FLOAT64", FieldType.DOUBLE) .put("INT32", FieldType.INT32) .put("INT64", FieldType.INT64) .put("STRING", FieldType.STRING)
[beam] branch master updated: [BEAM-6284] Improve error message on waitUntilFinish. (#8629)
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new ea32ab9 [BEAM-6284] Improve error message on waitUntilFinish. (#8629) ea32ab9 is described below commit ea32ab940453a1800dffce0833d732b7246deafa Author: Mikhail Gryzykhin <12602502+arda...@users.noreply.github.com> AuthorDate: Thu May 23 21:04:16 2019 -0700 [BEAM-6284] Improve error message on waitUntilFinish. (#8629) * Improve error message on waitUntilFinish. Allow for infinite wait. --- .../beam/runners/dataflow/DataflowPipelineJob.java | 229 + .../runners/dataflow/DataflowPipelineJobTest.java | 40 +++- 2 files changed, 177 insertions(+), 92 deletions(-) diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java index 049a904..c592425 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java @@ -38,6 +38,7 @@ import java.util.concurrent.atomic.AtomicReference; import javax.annotation.Nullable; import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.runners.dataflow.util.MonitoringUtil; +import org.apache.beam.runners.dataflow.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.extensions.gcp.util.BackOffAdapter; import org.apache.beam.sdk.metrics.MetricResults; @@ -53,6 +54,7 @@ import org.slf4j.LoggerFactory; /** A DataflowPipelineJob represents a job submitted to Dataflow using {@link DataflowRunner}. */ public class DataflowPipelineJob implements PipelineResult { + private static final Logger LOG = LoggerFactory.getLogger(DataflowPipelineJob.class); /** The id for the job. */ @@ -92,6 +94,8 @@ public class DataflowPipelineJob implements PipelineResult { static final Duration STATUS_POLLING_INTERVAL = Duration.standardSeconds(2); + static final Duration DEFAULT_MAX_BACKOFF = Duration.standardMinutes(2); + static final double DEFAULT_BACKOFF_EXPONENT = 1.5; /** The amount of polling retries for job status and messages information. */ @@ -103,7 +107,9 @@ public class DataflowPipelineJob implements PipelineResult { FluentBackoff.DEFAULT .withInitialBackoff(MESSAGES_POLLING_INTERVAL) .withMaxRetries(MESSAGES_POLLING_RETRIES) - .withExponent(DEFAULT_BACKOFF_EXPONENT); + .withExponent(DEFAULT_BACKOFF_EXPONENT) + .withMaxBackoff(DEFAULT_MAX_BACKOFF); + protected static final FluentBackoff STATUS_BACKOFF_FACTORY = FluentBackoff.DEFAULT .withInitialBackoff(STATUS_POLLING_INTERVAL) @@ -238,6 +244,16 @@ public class DataflowPipelineJob implements PipelineResult { duration, messageHandler, sleeper, nanoClock, new MonitoringUtil(dataflowClient)); } + private static BackOff getMessagesBackoff(Duration duration) { +FluentBackoff factory = MESSAGES_BACKOFF_FACTORY; + +if (!duration.isShorterThan(Duration.ZERO)) { + factory = factory.withMaxCumulativeBackoff(duration); +} + +return BackOffAdapter.toGcpBackOff(factory.backoff()); + } + /** * Waits until the pipeline finishes and returns the final status. * @@ -261,96 +277,128 @@ public class DataflowPipelineJob implements PipelineResult { MonitoringUtil monitor) throws IOException, InterruptedException { -BackOff backoff; -if (!duration.isLongerThan(Duration.ZERO)) { - backoff = BackOffAdapter.toGcpBackOff(MESSAGES_BACKOFF_FACTORY.backoff()); -} else { - backoff = - BackOffAdapter.toGcpBackOff( - MESSAGES_BACKOFF_FACTORY.withMaxCumulativeBackoff(duration).backoff()); -} +BackOff backoff = getMessagesBackoff(duration); // This function tracks the cumulative time from the *first request* to enforce the wall-clock // limit. Any backoff instance could, at best, track the the time since the first attempt at a // given request. Thus, we need to track the cumulative time ourselves. long startNanos = nanoClock.nanoTime(); -State state; +State state = State.UNKNOWN; +Exception exception; do { - // Get the state of the job before listing messages. This ensures we always fetch job - // messages after the job finishes to ensure we have all them. - state = - getStateWithRetries( - BackOffAdapter.toGcpBackOff(STATUS_BACKOFF_FACTORY.withMaxRetries(0).backoff()), - sleeper); - boolean hasError = state == State.U
[beam] branch master updated: [BEAM-7154] Updating Go SDK errors (Part 3)
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 7c92fe0 [BEAM-7154] Updating Go SDK errors (Part 3) new 4f9361b Merge pull request #8560 from youngoli/beam7154-2 7c92fe0 is described below commit 7c92fe0bc3ba73320ee26b6323eb01884381afcc Author: Daniel Oliveira AuthorDate: Sun May 12 20:51:46 2019 -0700 [BEAM-7154] Updating Go SDK errors (Part 3) --- sdks/go/pkg/beam/core/funcx/fn.go | 10 +- sdks/go/pkg/beam/core/funcx/signature.go | 15 ++- sdks/go/pkg/beam/core/graph/bind.go| 60 + sdks/go/pkg/beam/core/graph/coder/coder.go | 13 +- sdks/go/pkg/beam/core/graph/coder/registry.go | 7 +- sdks/go/pkg/beam/core/graph/coder/varint.go| 2 +- sdks/go/pkg/beam/core/graph/edge.go| 41 -- sdks/go/pkg/beam/core/graph/fn.go | 16 +-- sdks/go/pkg/beam/core/graph/graph.go | 7 +- sdks/go/pkg/beam/core/runtime/coderx/float.go | 5 +- sdks/go/pkg/beam/core/runtime/coderx/varint.go | 9 +- sdks/go/pkg/beam/core/runtime/exec/coder.go| 3 +- sdks/go/pkg/beam/core/runtime/exec/cogbk.go| 4 +- sdks/go/pkg/beam/core/runtime/exec/combine.go | 23 ++-- sdks/go/pkg/beam/core/runtime/exec/combine_test.go | 3 +- sdks/go/pkg/beam/core/runtime/exec/datasink.go | 3 +- sdks/go/pkg/beam/core/runtime/exec/datasource.go | 17 +-- sdks/go/pkg/beam/core/runtime/exec/fn.go | 15 ++- sdks/go/pkg/beam/core/runtime/exec/input.go| 3 +- sdks/go/pkg/beam/core/runtime/exec/pardo.go| 9 +- sdks/go/pkg/beam/core/runtime/exec/plan.go | 11 +- sdks/go/pkg/beam/core/runtime/exec/translate.go| 33 ++--- sdks/go/pkg/beam/core/runtime/exec/unit_test.go| 12 +- sdks/go/pkg/beam/core/runtime/exec/util.go | 5 +- sdks/go/pkg/beam/core/runtime/graphx/coder.go | 28 ++-- sdks/go/pkg/beam/core/runtime/graphx/dataflow.go | 27 ++-- sdks/go/pkg/beam/core/runtime/graphx/serialize.go | 149 ++--- sdks/go/pkg/beam/core/runtime/graphx/translate.go | 3 +- sdks/go/pkg/beam/core/runtime/harness/datamgr.go | 12 +- sdks/go/pkg/beam/core/runtime/harness/harness.go | 7 +- sdks/go/pkg/beam/core/runtime/harness/logging.go | 3 +- sdks/go/pkg/beam/core/runtime/harness/session.go | 13 +- sdks/go/pkg/beam/core/runtime/harness/statemgr.go | 14 +- sdks/go/pkg/beam/core/runtime/pipelinex/replace.go | 3 +- sdks/go/pkg/beam/core/runtime/symbols.go | 4 +- sdks/go/pkg/beam/core/typex/fulltype.go| 12 +- sdks/go/pkg/beam/core/util/dot/dot.go | 5 +- sdks/go/pkg/beam/core/util/hooks/hooks.go | 12 +- sdks/go/pkg/beam/core/util/ioutilx/read.go | 3 +- sdks/go/pkg/beam/core/util/protox/any.go | 9 +- sdks/go/pkg/beam/core/util/protox/base64.go| 4 +- sdks/go/pkg/beam/core/util/reflectx/call.go| 4 +- sdks/go/pkg/beam/core/util/reflectx/json.go| 5 +- sdks/go/pkg/beam/core/util/symtab/symtab.go| 15 ++- 44 files changed, 376 insertions(+), 282 deletions(-) diff --git a/sdks/go/pkg/beam/core/funcx/fn.go b/sdks/go/pkg/beam/core/funcx/fn.go index 48129a5..c924782 100644 --- a/sdks/go/pkg/beam/core/funcx/fn.go +++ b/sdks/go/pkg/beam/core/funcx/fn.go @@ -16,12 +16,12 @@ package funcx import ( - "errors" "fmt" "reflect" "github.com/apache/beam/sdks/go/pkg/beam/core/typex" "github.com/apache/beam/sdks/go/pkg/beam/core/util/reflectx" + "github.com/apache/beam/sdks/go/pkg/beam/internal/errors" ) // Note that we can't tell the difference between K, V and V, S before binding. @@ -250,7 +250,7 @@ func New(fn reflectx.Func) (*Fn, error) { case IsReIter(t): kind = FnReIter default: - return nil, fmt.Errorf("bad parameter type for %s: %v", fn.Name(), t) + return nil, errors.Errorf("bad parameter type for %s: %v", fn.Name(), t) } param = append(param, FnParam{Kind: kind, T: t}) @@ -269,7 +269,7 @@ func New(fn reflectx.Func) (*Fn, error) { case typex.IsContainer(t), typex.IsConcrete(t), typex.IsUniversal(t): kind = RetValue default: - return nil, fmt.Errorf("bad return type for %s: %v", fn.Name(), t) + return nil, errors.Errorf("bad return type for %s: %v", fn.Name(), t) } ret = append(ret, ReturnParam{Kind: kind, T: t}) @@ -314,14 +314,14 @@ func valida
[beam] branch master updated (c52935f -> 7b1abc9)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from c52935f Merge pull request #8342: [BEAM-7103] Adding AvroCoderTranslator for cross-language data transfer new 568c3db upgrade avatica to 1.14.0 new e2c9571 upgrade to avatica 1.15.0 new 7b1abc9 Merge pull request #8566 from vectorijk/upgrade-avatica The 21501 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: sdks/java/extensions/sql/build.gradle | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (b6f22ef -> 1715194)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from b6f22ef [BEAM-7269] remove StateSpec from hashCode of SimpleStateTag new 6c1d448 Small doc clean up new b4f1e37 [SQL] SpotlessApply for minor docs PR #8567 new 1715194 Merge pull request #8567 from akedin/minor-docs-cleanup The 21444 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/impl/rel/BeamSortRel.java | 19 ++- 1 file changed, 10 insertions(+), 9 deletions(-)
[beam] branch master updated: [sql] fix non return bug.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 92517ca [sql] fix non return bug. new 2e3f921 Merge pull request #8532 from amaliujia/rw-fix_join_bug 92517ca is described below commit 92517ca02fef76659798624305689bf7fcde664a Author: amaliujia AuthorDate: Wed May 8 14:11:52 2019 -0700 [sql] fix non return bug. --- .../java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java index fe066e2..08a1a33 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java @@ -503,7 +503,7 @@ public class BeamJoinRel extends Join implements BeamRelNode { return schema.getField(((RexInputRef) rexNode).getIndex() - leftRowColumnCount); } else if (rexNode instanceof RexFieldAccess) { // need to extract field of Struct/Row. - getFieldBasedOnRexFieldAccess(schema, (RexFieldAccess) rexNode, leftRowColumnCount); + return getFieldBasedOnRexFieldAccess(schema, (RexFieldAccess) rexNode, leftRowColumnCount); } throw new UnsupportedOperationException("Does not support " + rexNode.getType() + " in JOIN.");
[beam] branch master updated: [SQL] Refactor BeamSqlEnv
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 8135e39 [SQL] Refactor BeamSqlEnv new 79a4637 Merge pull request #8523 from akedin/refactor-sqlenv 8135e39 is described below commit 8135e39952a117152f8709daa27a9fabb66567ad Author: akedin AuthorDate: Tue May 7 15:19:02 2019 -0700 [SQL] Refactor BeamSqlEnv --- .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 10 +- .../beam/sdk/extensions/sql/SqlTransform.java | 18 +- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 207 - .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 8 +- .../extensions/sql/impl/parser/BeamDDLTest.java| 6 +- 5 files changed, 136 insertions(+), 113 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java index d28ac2f..5e44c6c 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java @@ -39,11 +39,11 @@ public class BeamSqlCli { public BeamSqlCli metaStore(MetaStore metaStore, boolean autoLoadUdfUdaf) { this.metaStore = metaStore; -this.env = -BeamSqlEnv.builder() -.setInitializeTableProvider(metaStore) -.loadUdfUdafFromProvider() -.build(); +BeamSqlEnv.BeamSqlEnvBuilder builder = BeamSqlEnv.builder(metaStore); +if (autoLoadUdfUdaf) { + builder.autoLoadUserDefinedFunctions(); +} +this.env = builder.build(); return this; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java index f9d7700..e45daca 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java @@ -64,7 +64,7 @@ import org.apache.beam.vendor.guava.v20_0.com.google.common.collect.ImmutableMap * PCollection outputTableA = inputTableA.apply( *SqlTransform *.query(sql1) - *.registerUdf("MY_FUNC", MY_FUNC.class, "FUNC"); + *.addUdf("MY_FUNC", MY_FUNC.class, "FUNC"); * * //run a JOIN with one table from TextIO, and one table from another query * PCollection outputTableB = @@ -99,19 +99,20 @@ public abstract class SqlTransform extends PTransform> @Override public PCollection expand(PInput input) { BeamSqlEnvBuilder sqlEnvBuilder = -BeamSqlEnv.builder() -.setInitializeTableProvider( -new ReadOnlyTableProvider(PCOLLECTION_NAME, toTableMap(input))); +BeamSqlEnv.builder(new ReadOnlyTableProvider(PCOLLECTION_NAME, toTableMap(input))); + tableProviderMap().forEach(sqlEnvBuilder::addSchema); + if (defaultTableProvider() != null) { sqlEnvBuilder.setCurrentSchema(defaultTableProvider()); } // TODO: validate duplicate functions. -sqlEnvBuilder.loadBeamBuiltinFunctions(); +sqlEnvBuilder.autoLoadBuiltinFunctions(); registerFunctions(sqlEnvBuilder); + if (autoUdfUdafLoad()) { - sqlEnvBuilder.loadUdfUdafFromProvider(); + sqlEnvBuilder.autoLoadUserDefinedFunctions(); } sqlEnvBuilder.setQueryPlannerClassName( @@ -142,10 +143,9 @@ public abstract class SqlTransform extends PTransform> private void registerFunctions(BeamSqlEnvBuilder sqlEnvBuilder) { udfDefinitions() -.forEach(udf -> sqlEnvBuilder.registerUdf(udf.udfName(), udf.clazz(), udf.methodName())); +.forEach(udf -> sqlEnvBuilder.addUdf(udf.udfName(), udf.clazz(), udf.methodName())); -udafDefinitions() -.forEach(udaf -> sqlEnvBuilder.registerUdaf(udaf.udafName(), udaf.combineFn())); +udafDefinitions().forEach(udaf -> sqlEnvBuilder.addUdaf(udaf.udafName(), udaf.combineFn())); } /** diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnv.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnv.java index df6a2e4..02b3e69 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnv.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnv.java @@ -17,7 +17,8 @@ */ package org.apache.beam.sdk.extensions.sql.impl; -import java.lang.reflect.InvocationTargetException; +import static org.apache.beam.vendor.guava.v20_0.com.google.common.base.Preconditions
[beam] branch master updated: [SQL] Upgrade DataCatalog client to 0.4.0-alpha
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 6a45d90 [SQL] Upgrade DataCatalog client to 0.4.0-alpha new 8e14f2f Merge pull request #8527 from akedin/upgrade-dc-040 6a45d90 is described below commit 6a45d90cda5b83d9f4f552da85b9fb521a7469b4 Author: akedin AuthorDate: Tue May 7 17:29:33 2019 -0700 [SQL] Upgrade DataCatalog client to 0.4.0-alpha --- buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 066c64c..b0794fd 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -368,7 +368,7 @@ class BeamModulePlugin implements Plugin { def cassandra_driver_version = "3.6.0" def generated_grpc_beta_version = "0.44.0" def generated_grpc_ga_version = "1.43.0" -def generated_grpc_dc_beta_version = "0.1.0-alpha" +def generated_grpc_dc_beta_version = "0.4.0-alpha" def google_auth_version = "0.12.0" def google_clients_version = "1.27.0" def google_cloud_bigdataoss_version = "1.9.16"
[beam] branch master updated: [SQL][Fix] Fix DataCatalog MAP type
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 9cd0d56 [SQL][Fix] Fix DataCatalog MAP type new 52481a8 Merge pull request #8524 from akedin/fix-datacatalog-type-map 9cd0d56 is described below commit 9cd0d56e7da547bdee278c896f8869670e8989c4 Author: akedin AuthorDate: Tue May 7 16:01:07 2019 -0700 [SQL][Fix] Fix DataCatalog MAP type --- .../beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java index 80a5ece..69288a8 100644 --- a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java +++ b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java @@ -44,7 +44,7 @@ class SchemaUtils { .put("STRING", FieldType.STRING) .put("TIME", FieldType.logicalType(new CalciteUtils.TimeType())) .put("TIMESTAMP", FieldType.DATETIME) - .put("MAP", FieldType.map(FieldType.STRING, FieldType.STRING)) + .put("MAP", FieldType.map(FieldType.STRING, FieldType.STRING)) .build(); /** Convert DataCatalog schema to Beam schema. */
[beam] branch master updated: [SQL] Remove TYPE_ prefix from DataCatalogTableProvider
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 90677a1 [SQL] Remove TYPE_ prefix from DataCatalogTableProvider new 21700d7 Merge pull request #8520 from akedin/remove-type-prefix 90677a1 is described below commit 90677a1b954a74fe1f19684556ac517feb271299 Author: akedin AuthorDate: Tue May 7 14:37:29 2019 -0700 [SQL] Remove TYPE_ prefix from DataCatalogTableProvider --- .../sql/meta/provider/datacatalog/SchemaUtils.java | 28 ++ .../beam/sdk/io/gcp/bigquery/BigQueryUtils.java| 3 ++- 2 files changed, 15 insertions(+), 16 deletions(-) diff --git a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java index c28c820..80a5ece 100644 --- a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java +++ b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/SchemaUtils.java @@ -33,20 +33,18 @@ class SchemaUtils { private static final Map FIELD_TYPES = ImmutableMap.builder() - .put("TYPE_BOOL", FieldType.BOOLEAN) - .put("TYPE_BYTES", FieldType.BYTES) - .put("TYPE_DATE", FieldType.logicalType(new CalciteUtils.DateType())) - .put("TYPE_DATETIME", FieldType.DATETIME) - .put("TYPE_DOUBLE", FieldType.DOUBLE) - .put("TYPE_FLOAT", FieldType.DOUBLE) - .put("TYPE_INT32", FieldType.INT32) - .put("TYPE_INT64", FieldType.INT64) - .put("TYPE_STRING", FieldType.STRING) - .put("TYPE_TIME", FieldType.logicalType(new CalciteUtils.TimeType())) - .put("TYPE_TIMESTAMP", FieldType.DATETIME) - .put( - "TYPE_MAP", - FieldType.map(FieldType.STRING, FieldType.STRING)) + .put("BOOL", FieldType.BOOLEAN) + .put("BYTES", FieldType.BYTES) + .put("DATE", FieldType.logicalType(new CalciteUtils.DateType())) + .put("DATETIME", FieldType.DATETIME) + .put("DOUBLE", FieldType.DOUBLE) + .put("FLOAT", FieldType.DOUBLE) + .put("INT32", FieldType.INT32) + .put("INT64", FieldType.INT64) + .put("STRING", FieldType.STRING) + .put("TIME", FieldType.logicalType(new CalciteUtils.TimeType())) + .put("TIMESTAMP", FieldType.DATETIME) + .put("MAP", FieldType.map(FieldType.STRING, FieldType.STRING)) .build(); /** Convert DataCatalog schema to Beam schema. */ @@ -87,7 +85,7 @@ class SchemaUtils { return FIELD_TYPES.get(dcFieldType); } -if ("TYPE_STRUCT".equals(dcFieldType)) { +if ("STRUCT".equals(dcFieldType)) { Schema structSchema = fromColumnsList(column.getSubcolumnsList()); return FieldType.row(structSchema); } diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java index 2c1e138..bd1fda3 100644 --- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java +++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryUtils.java @@ -394,7 +394,8 @@ public class BigQueryUtils { case MAP: throw new RuntimeException("Does not support converting MAP type value"); default: -throw new RuntimeException("Does not support converting unknown type value"); +throw new RuntimeException( +"Does not support converting unknown type value: " + beamFieldTypeName); } }
[beam] branch master updated: [BEAM-5644] make Planner configurable.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new d2851bf [BEAM-5644] make Planner configurable. new c25b34a Merge pull request #7745 from amaliujia/rw-configurable_planner d2851bf is described below commit d2851bff01796ac678a4e8f2b2c971c2dc0a1a13 Author: amaliujia AuthorDate: Tue Feb 5 20:16:06 2019 -0800 [BEAM-5644] make Planner configurable. --- .../apache/beam/sdk/extensions/sql/BeamSqlCli.java | 10 +- .../beam/sdk/extensions/sql/SqlTransform.java | 29 ++- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 254 ++--- .../sql/impl/BeamSqlPipelineOptions.java | 32 +++ .../sql/impl/BeamSqlPipelineOptionsRegistrar.java | 33 +++ ...mQueryPlanner.java => CalciteQueryPlanner.java} | 31 +-- .../beam/sdk/extensions/sql/impl/QueryPlanner.java | 33 +++ .../sql/impl/SqlConversionException.java | 30 +++ .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 26 ++- .../extensions/sql/impl/parser/BeamDDLTest.java| 8 +- 10 files changed, 365 insertions(+), 121 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java index 6d2c167..d28ac2f 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/BeamSqlCli.java @@ -39,11 +39,11 @@ public class BeamSqlCli { public BeamSqlCli metaStore(MetaStore metaStore, boolean autoLoadUdfUdaf) { this.metaStore = metaStore; -this.env = BeamSqlEnv.withTableProvider(metaStore); -if (autoLoadUdfUdaf) { - env.loadUdfUdafFromProvider(); -} - +this.env = +BeamSqlEnv.builder() +.setInitializeTableProvider(metaStore) +.loadUdfUdafFromProvider() +.build(); return this; } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java index f5613c9..f9d7700 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/SqlTransform.java @@ -25,8 +25,11 @@ import java.util.Map; import javax.annotation.Nullable; import org.apache.beam.sdk.annotations.Experimental; import org.apache.beam.sdk.extensions.sql.impl.BeamSqlEnv; +import org.apache.beam.sdk.extensions.sql.impl.BeamSqlEnv.BeamSqlEnvBuilder; +import org.apache.beam.sdk.extensions.sql.impl.BeamSqlPipelineOptions; import org.apache.beam.sdk.extensions.sql.impl.rel.BeamSqlRelUtils; import org.apache.beam.sdk.extensions.sql.impl.schema.BeamPCollectionTable; +import org.apache.beam.sdk.extensions.sql.meta.provider.ReadOnlyTableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.TableProvider; import org.apache.beam.sdk.transforms.Combine; import org.apache.beam.sdk.transforms.PTransform; @@ -95,19 +98,26 @@ public abstract class SqlTransform extends PTransform> @Override public PCollection expand(PInput input) { -BeamSqlEnv sqlEnv = BeamSqlEnv.readOnly(PCOLLECTION_NAME, toTableMap(input)); -tableProviderMap().forEach(sqlEnv::addSchema); +BeamSqlEnvBuilder sqlEnvBuilder = +BeamSqlEnv.builder() +.setInitializeTableProvider( +new ReadOnlyTableProvider(PCOLLECTION_NAME, toTableMap(input))); +tableProviderMap().forEach(sqlEnvBuilder::addSchema); if (defaultTableProvider() != null) { - sqlEnv.setCurrentSchema(defaultTableProvider()); + sqlEnvBuilder.setCurrentSchema(defaultTableProvider()); } // TODO: validate duplicate functions. -sqlEnv.loadBeamBuiltinFunctions(); -registerFunctions(sqlEnv); +sqlEnvBuilder.loadBeamBuiltinFunctions(); +registerFunctions(sqlEnvBuilder); if (autoUdfUdafLoad()) { - sqlEnv.loadUdfUdafFromProvider(); + sqlEnvBuilder.loadUdfUdafFromProvider(); } +sqlEnvBuilder.setQueryPlannerClassName( + input.getPipeline().getOptions().as(BeamSqlPipelineOptions.class).getPlannerName()); + +BeamSqlEnv sqlEnv = sqlEnvBuilder.build(); return BeamSqlRelUtils.toPCollection(input.getPipeline(), sqlEnv.parseQuery(queryString())); } @@ -130,11 +140,12 @@ public abstract class SqlTransform extends PTransform> return tables.build(); } - private void registerFunctions(BeamSqlEnv sqlEnv) { + private void registerFunctions(BeamSqlEnvBuilder sqlEnvBuilder) { udfDefinitions() -.forEach(udf -> sqlEnv.registerUdf(udf.udfName(), udf.clazz(), ud
[beam] branch master updated: Move schema assignment onto Create builder
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new b343782 Move schema assignment onto Create builder new 2f821dc Merge pull request #8479 from TheNeuralBit/empty-value-schema b343782 is described below commit b34378257501494a4e26c64f7c99da3abaaa718c Author: Brian Hulette AuthorDate: Thu May 2 14:44:59 2019 -0700 Move schema assignment onto Create builder Fixes a bug where an empty BeamValuesResl throws an exception because it cannot create a default coder. --- .../beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java | 2 +- .../beam/sdk/extensions/sql/impl/rel/BeamValuesRelTest.java | 13 + 2 files changed, 14 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java index b681738..9799ed8 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java @@ -82,7 +82,7 @@ public class BeamValuesRel extends Values implements BeamRelNode { Schema schema = CalciteUtils.toSchema(getRowType()); List rows = tuples.stream().map(tuple -> tupleToRow(schema, tuple)).collect(toList()); - return pinput.getPipeline().begin().apply(Create.of(rows)).setRowSchema(schema); + return pinput.getPipeline().begin().apply(Create.of(rows).withRowSchema(schema)); } } diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRelTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRelTest.java index cdc2eb2..0dc8e26 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRelTest.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRelTest.java @@ -91,4 +91,17 @@ public class BeamValuesRelTest extends BaseRelTest { .getRows()); pipeline.run(); } + + @Test + public void testValues_selectEmpty() throws Exception { +String sql = "select 1, '1' FROM string_table WHERE false"; +PCollection rows = compilePipeline(sql, pipeline); +PAssert.that(rows) +.containsInAnyOrder( +TestUtils.RowsBuilder.of( +Schema.FieldType.INT32, "EXPR$0", +Schema.FieldType.STRING, "EXPR$1") +.getRows()); +pipeline.run(); + } }
[beam] branch master updated: [BEAM-7166] Add more checks on join condition.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new a0515de [BEAM-7166] Add more checks on join condition. new 12b0493 Merge pull request #8421 from amaliujia/rw-more_checks_on_join_condition a0515de is described below commit a0515de0f593af234767f83f161699456b682cf6 Author: amaliujia AuthorDate: Fri Apr 26 21:41:58 2019 -0700 [BEAM-7166] Add more checks on join condition. --- .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 15 - .../impl/rel/BeamJoinRelBoundedVsBoundedTest.java | 65 ++ 2 files changed, 79 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java index 1696329..fe066e2 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java @@ -566,7 +566,12 @@ public class BeamJoinRel extends Join implements BeamRelNode { private Pair extractJoinPairOfRexNodes(RexCall rexCall) { if (!rexCall.getOperator().getName().equals("=")) { - throw new UnsupportedOperationException("Non equi-join is not supported!"); + throw new UnsupportedOperationException("Non equi-join is not supported"); +} + +if (isIllegalJoinConjunctionClause(rexCall)) { + throw new UnsupportedOperationException( + "Only support column reference or struct field access in conjunction clause"); } int leftIndex = getColumnIndex(rexCall.getOperands().get(0)); @@ -578,6 +583,14 @@ public class BeamJoinRel extends Join implements BeamRelNode { } } + // Only support {RexInputRef | RexFieldAccess} = {RexInputRef | RexFieldAccess} + private boolean isIllegalJoinConjunctionClause(RexCall rexCall) { +return (!(rexCall.getOperands().get(0) instanceof RexInputRef) +&& !(rexCall.getOperands().get(0) instanceof RexFieldAccess)) +|| (!(rexCall.getOperands().get(1) instanceof RexInputRef) +&& !(rexCall.getOperands().get(1) instanceof RexFieldAccess)); + } + private int getColumnIndex(RexNode rexNode) { if (rexNode instanceof RexInputRef) { return ((RexInputRef) rexNode).getIndex(); diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java index 162b0ef..d8e8a61 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java @@ -24,13 +24,16 @@ import org.apache.beam.sdk.testing.PAssert; import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.Row; +import org.hamcrest.core.StringContains; import org.junit.BeforeClass; import org.junit.Rule; import org.junit.Test; +import org.junit.rules.ExpectedException; /** Bounded + Bounded Test for {@code BeamJoinRel}. */ public class BeamJoinRelBoundedVsBoundedTest extends BaseRelTest { @Rule public final TestPipeline pipeline = TestPipeline.create(); + @Rule public ExpectedException thrown = ExpectedException.none(); public static final TestBoundedTable ORDER_DETAILS1 = TestBoundedTable.of( @@ -227,6 +230,68 @@ public class BeamJoinRelBoundedVsBoundedTest extends BaseRelTest { pipeline.run(); } + @Test + public void testException_join_condition1() throws Exception { +String sql = +"SELECT * " ++ "FROM ORDER_DETAILS1 o1" ++ " JOIN ORDER_DETAILS2 o2" ++ " on " ++ " o1.order_id = o2.site_id OR o1.price = o2.site_id"; + +thrown.expect(UnsupportedOperationException.class); +thrown.expectMessage(StringContains.containsString("Operator OR")); +compilePipeline(sql, pipeline); +pipeline.run(); + } + + @Test + public void testException_join_condition2() throws Exception { +String sql = +"SELECT * " ++ "FROM ORDER_DETAILS1 o1" ++ " JOIN ORDER_DETAILS2 o2" ++ " on " ++ " o1.order_id = o2.site_id AND o1.price > o2.site_id"; + +thrown.expect(UnsupportedOperationException.class); +thrown.expectMessage(StringC
[beam] branch master updated: [BEAM-7072][SQL][Nexmark] Disable Query5
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 4c654122 [BEAM-7072][SQL][Nexmark] Disable Query5 new ef67378 Merge pull request #8431 from akedin/disable-sqlquery5 4c654122 is described below commit 4c65412214802e7467cd38ea71b711418a2670fd Author: akedin AuthorDate: Mon Apr 29 13:56:39 2019 -0700 [BEAM-7072][SQL][Nexmark] Disable Query5 --- .../org/apache/beam/sdk/nexmark/NexmarkLauncher.java| 17 + 1 file changed, 13 insertions(+), 4 deletions(-) diff --git a/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java b/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java index 0b03166..419d151 100644 --- a/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java +++ b/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java @@ -81,7 +81,6 @@ import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery0; import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery1; import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery2; import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery3; -import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery5; import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery7; import org.apache.beam.sdk.testing.PAssert; import org.apache.beam.sdk.testutils.metrics.MetricsReader; @@ -1241,9 +1240,19 @@ public class NexmarkLauncher { .put( NexmarkQueryName.LOCAL_ITEM_SUGGESTION, new NexmarkQuery(configuration, new SqlQuery3(configuration))) -.put( -NexmarkQueryName.HOT_ITEMS, -new NexmarkQuery(configuration, new SqlQuery5(configuration))) + +// SqlQuery5 is disabled for now, uses non-equi-joins, +// never worked right, was giving incorrect results. +// Gets rejected after PR/8301, causing failures. +// +// See: +// https://issues.apache.org/jira/browse/BEAM-7072 +// https://github.com/apache/beam/pull/8301 +// https://github.com/apache/beam/pull/8422#issuecomment-487676350 +// +//.put( +//NexmarkQueryName.HOT_ITEMS, +//new NexmarkQuery(configuration, new SqlQuery5(configuration))) .put( NexmarkQueryName.HIGHEST_BID, new NexmarkQuery(configuration, new SqlQuery7(configuration)))
[beam] branch master updated (270d8fe -> 2cb44a8)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 270d8fe Merge pull request #8401 Fix small bug in top documentation new a5f3a75 [BEAM-7070] JOIN condition should accept field access new 6622642 [sql] ignore Nexmark SQL queries that has non equal join. new 152c6e0 [sql] generalize RexInputRef and RexFieldAccess in JOIN. new 2cb44a8 Merge pull request #8301 from amaliujia/rw-improve_join_condition The 21140 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../sdk/extensions/sql/impl/rel/BeamJoinRel.java | 103 - .../sql/impl/transform/BeamJoinTransforms.java | 38 +++- .../sql/impl/utils/SerializableRexFieldAccess.java | 55 +++ ...kage-info.java => SerializableRexInputRef.java} | 17 +++- .../sql/impl/utils/SerializableRexNode.java| 50 ++ .../apache/beam/sdk/nexmark/queries/QueryTest.java | 2 + .../sdk/nexmark/queries/sql/SqlQuery5Test.java | 2 + 7 files changed, 240 insertions(+), 27 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/SerializableRexFieldAccess.java copy sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/{package-info.java => SerializableRexInputRef.java} (73%) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/SerializableRexNode.java
[beam] branch master updated: [SQL] Add Data Catalog Table Provider
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 9a99664 [SQL] Add Data Catalog Table Provider new 9c8a8dc Merge pull request #8349 from akedin/datacatalog-table-provider 9a99664 is described below commit 9a99664822a8fd79bb5491e206159fce064c9882 Author: akedin AuthorDate: Wed Apr 17 22:55:01 2019 -0700 [SQL] Add Data Catalog Table Provider --- .../org/apache/beam/gradle/BeamModulePlugin.groovy | 3 + sdks/java/extensions/sql/datacatalog/build.gradle | 63 +++ .../sql/example/BeamSqlDataCatalogExample.java | 103 + .../sdk/extensions/sql/example/package-info.java | 20 .../meta/provider/datacatalog/BigQueryUtils.java | 58 ++ .../datacatalog/DataCatalogClientAdapter.java | 94 .../datacatalog/DataCatalogPipelineOptions.java| 35 ++ .../datacatalog/DataCatalogTableProvider.java | 122 + .../sql/meta/provider/datacatalog/PubsubUtils.java | 53 + .../sql/meta/provider/datacatalog/SchemaUtils.java | 98 + .../sql/meta/provider/datacatalog/TableUtils.java | 59 ++ .../meta/provider/datacatalog/package-info.java| 20 .../beam/sdk/io/gcp/bigquery/BigQueryUtils.java| 4 +- settings.gradle| 2 + 14 files changed, 733 insertions(+), 1 deletion(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 74bc02f..6614954 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -350,6 +350,7 @@ class BeamModulePlugin implements Plugin { def cassandra_driver_version = "3.6.0" def generated_grpc_beta_version = "0.44.0" def generated_grpc_ga_version = "1.43.0" +def generated_grpc_dc_beta_version = "0.1.0-alpha" def google_auth_version = "0.12.0" def google_clients_version = "1.27.0" def google_cloud_bigdataoss_version = "1.9.16" @@ -442,6 +443,7 @@ class BeamModulePlugin implements Plugin { grpc_all: "io.grpc:grpc-all:$grpc_version", grpc_auth : "io.grpc:grpc-auth:$grpc_version", grpc_core : "io.grpc:grpc-core:$grpc_version", +grpc_google_cloud_datacatalog_v1beta1 : "com.google.api.grpc:grpc-google-cloud-datacatalog-v1beta1:$generated_grpc_dc_beta_version", grpc_google_cloud_pubsub_v1 : "com.google.api.grpc:grpc-google-cloud-pubsub-v1:$generated_grpc_ga_version", grpc_protobuf : "io.grpc:grpc-protobuf:$grpc_version", grpc_protobuf_lite : "io.grpc:grpc-protobuf-lite:$grpc_version", @@ -480,6 +482,7 @@ class BeamModulePlugin implements Plugin { powermock : "org.powermock:powermock-mockito-release-full:1.6.4", protobuf_java : "com.google.protobuf:protobuf-java:$protobuf_version", protobuf_java_util : "com.google.protobuf:protobuf-java-util:$protobuf_version", +proto_google_cloud_datacatalog_v1beta1 : "com.google.api.grpc:proto-google-cloud-datacatalog-v1beta1:$generated_grpc_dc_beta_version", proto_google_cloud_pubsub_v1: "com.google.api.grpc:proto-google-cloud-pubsub-v1:$generated_grpc_ga_version", proto_google_cloud_spanner_admin_database_v1: "com.google.api.grpc:proto-google-cloud-spanner-admin-database-v1:$google_cloud_spanner_version", proto_google_common_protos : "com.google.api.grpc:proto-google-common-protos:$proto_google_common_protos_version", diff --git a/sdks/java/extensions/sql/datacatalog/build.gradle b/sdks/java/extensions/sql/datacatalog/build.gradle new file mode 100644 index 000..9bf974a --- /dev/null +++ b/sdks/java/extensions/sql/datacatalog/build.gradle @@ -0,0 +1,63 @@ +import groovy.json.JsonOutput + +/* + * 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 o
[beam] branch master updated: [BEAM-7100] BeamValuesRel should accept empty tuples
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e99dd29 [BEAM-7100] BeamValuesRel should accept empty tuples new 3128cf5 Merge pull request #8339 from amaliujia/rw_empty_join_on_one_side e99dd29 is described below commit e99dd2970e1dc075c0bb3e553c5afb8a1f317b40 Author: amaliujia AuthorDate: Wed Apr 17 14:21:46 2019 -0700 [BEAM-7100] BeamValuesRel should accept empty tuples --- .../sdk/extensions/sql/impl/rel/BeamValuesRel.java | 6 --- .../impl/rel/BeamJoinRelBoundedVsBoundedTest.java | 54 ++ 2 files changed, 54 insertions(+), 6 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java index dbb5bd8..b681738 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamValuesRel.java @@ -80,14 +80,8 @@ public class BeamValuesRel extends Values implements BeamRelNode { BeamValuesRel.class.getSimpleName(), pinput); - if (tuples.isEmpty()) { -throw new IllegalStateException("Values with empty tuples!"); - } - Schema schema = CalciteUtils.toSchema(getRowType()); - List rows = tuples.stream().map(tuple -> tupleToRow(schema, tuple)).collect(toList()); - return pinput.getPipeline().begin().apply(Create.of(rows)).setRowSchema(schema); } } diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java index 3286042..162b0ef 100644 --- a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRelBoundedVsBoundedTest.java @@ -106,6 +106,60 @@ public class BeamJoinRelBoundedVsBoundedTest extends BaseRelTest { } @Test + public void testLeftOuterJoinWithEmptyTuplesOnRightSide() throws Exception { +String sql = +"SELECT * " ++ "FROM ORDER_DETAILS1 o1" ++ " LEFT OUTER JOIN (SELECT * FROM ORDER_DETAILS2 WHERE FALSE) o2" ++ " on " ++ " o1.order_id=o2.site_id AND o2.price=o1.site_id"; + +PCollection rows = compilePipeline(sql, pipeline); +pipeline.enableAbandonedNodeEnforcement(false); +PAssert.that(rows) +.containsInAnyOrder( +TestUtils.RowsBuilder.of( +Schema.builder() +.addField("order_id", Schema.FieldType.INT32) +.addField("site_id", Schema.FieldType.INT32) +.addField("price", Schema.FieldType.INT32) +.addNullableField("order_id0", Schema.FieldType.INT32) +.addNullableField("site_id0", Schema.FieldType.INT32) +.addNullableField("price0", Schema.FieldType.INT32) +.build()) +.addRows( +1, 2, 3, null, null, null, 2, 3, 3, null, null, null, 3, 4, 5, null, null, null) +.getRows()); +pipeline.run(); + } + + @Test + public void testInnerJoinWithEmptyTuplesOnRightSide() throws Exception { +String sql = +"SELECT * " ++ "FROM ORDER_DETAILS1 o1" ++ " INNER JOIN (SELECT * FROM ORDER_DETAILS2 WHERE FALSE) o2" ++ " on " ++ " o1.order_id=o2.site_id AND o2.price=o1.site_id"; + +PCollection rows = compilePipeline(sql, pipeline); +pipeline.enableAbandonedNodeEnforcement(false); +PAssert.that(rows) +.containsInAnyOrder( +TestUtils.RowsBuilder.of( +Schema.builder() +.addField("order_id", Schema.FieldType.INT32) +.addField("site_id", Schema.FieldType.INT32) +.addField("price", Schema.FieldType.INT32) +.addNullableField("order_id0", Schema.FieldType.INT32) +.addNullableField("site_id0", Schema.FieldType.INT32) +.addNullableField("price0", Schema.FieldType.INT32) +.build()) +.getRows()); +pipeline.run(); + } + + @Test public void testRightOuterJoin() throws Exception { String sql = "SELECT * "
[beam] branch master updated: [SQL] Move HCatalogTableProvider into its own module
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new c7fad9e [SQL] Move HCatalogTableProvider into its own module new 2cd2677 Merge pull request #8345 from akedin/move-hcatalogtableprovider c7fad9e is described below commit c7fad9e36c4e78bce9d8f7fb2b7aa6916a1b3940 Author: akedin AuthorDate: Wed Apr 17 22:53:32 2019 -0700 [SQL] Move HCatalogTableProvider into its own module --- sdks/java/extensions/sql/build.gradle | 16 - sdks/java/extensions/sql/hcatalog/build.gradle | 40 ++ .../meta/provider/hcatalog/DatabaseProvider.java | 0 .../sql/meta/provider/hcatalog/HCatalogTable.java | 0 .../provider/hcatalog/HCatalogTableProvider.java | 0 .../sql/meta/provider/hcatalog/package-info.java | 0 .../provider/hcatalog}/BeamSqlHiveSchemaTest.java | 4 +-- settings.gradle| 2 ++ 8 files changed, 44 insertions(+), 18 deletions(-) diff --git a/sdks/java/extensions/sql/build.gradle b/sdks/java/extensions/sql/build.gradle index 703bc76..2039389 100644 --- a/sdks/java/extensions/sql/build.gradle +++ b/sdks/java/extensions/sql/build.gradle @@ -60,8 +60,6 @@ configurations { def calcite_version = "1.19.0" def avatica_version = "1.13.0" -def hive_version = "2.1.0" -def netty_version = "4.1.30.Final" dependencies { javacc "net.java.dev.javacc:javacc:4.0" @@ -85,7 +83,6 @@ dependencies { shadow "com.alibaba:fastjson:1.2.49" shadow "com.jayway.jsonpath:json-path:2.4.0" shadow project(path: ":beam-runners-direct-java", configuration: "shadow") - provided project(path: ":beam-sdks-java-io-hcatalog", configuration: "shadow") provided project(path: ":beam-sdks-java-io-kafka", configuration: "shadow") provided project(path: ":beam-sdks-java-io-google-cloud-platform", configuration: "shadow") provided library.java.kafka_clients @@ -96,18 +93,6 @@ dependencies { shadowTest library.java.mockito_core shadowTest library.java.quickcheck_core - // Needed for HCatalogTableProvider tests, - // they use HCat* types - shadowTest "io.netty:netty-all:$netty_version" - shadowTest("org.apache.hive.hcatalog:hive-hcatalog-core:$hive_version") { -// Hive brings full Calcite 1.6 + Avatica with JDBC driver which -// gets registered and gets started instead of ours, -// and a ton of other stuff. -exclude group: "org.apache.calcite", module:"calcite-avatica" -exclude group: "com.google.guava", module: "guava" -exclude group: "com.google.protobuf", module: "protobuf-java" - } - // Dependencies that we don't directly reference permitUnusedDeclared "com.jayway.jsonpath:json-path:2.4.0" permitUnusedDeclared library.java.jackson_dataformat_yaml @@ -209,7 +194,6 @@ task integrationTest(type: Test) { .sourceSets .test .runtimeClasspath - .filter { it.name != "hive-exec-${hive_version}.jar" } testClassesDirs = files(project(":beam-sdks-java-extensions-sql").sourceSets.test.output.classesDirs) useJUnit { } } diff --git a/sdks/java/extensions/sql/hcatalog/build.gradle b/sdks/java/extensions/sql/hcatalog/build.gradle new file mode 100644 index 000..1aedb02 --- /dev/null +++ b/sdks/java/extensions/sql/hcatalog/build.gradle @@ -0,0 +1,40 @@ +import groovy.json.JsonOutput + +/* + * 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. + */ + +plugins { id 'org.apache.beam.module' } + +applyJavaNature() + +def hive_version = "2.1.0" +def netty_version = "4.1.30.Final" + +dependencies { + provided project(path: ":beam-sdks-java-extensions-sql", configuration: "shadow") + provided project(path: ":beam-sdks-java-io-hcatalog", configuration: "shadow") + + // Needed for HCatalog
[beam] branch master updated: [SQL] Make BigQuery schema conversion order-aware
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new fe327ef [SQL] Make BigQuery schema conversion order-aware new 120394b Merge pull request #8193 from akedin/bq-schema-order fe327ef is described below commit fe327ef97acc92f31711a242685710c5dfbc8249 Author: akedin AuthorDate: Mon Apr 1 15:48:51 2019 -0700 [SQL] Make BigQuery schema conversion order-aware --- .../{BeamBigQueryTable.java => BigQueryTable.java} | 30 +++--- .../provider/bigquery/BigQueryTableProvider.java | 6 + .../bigquery/BigQueryTableProviderTest.java| 6 ++--- .../beam/sdk/io/gcp/bigquery/BigQueryUtils.java| 11 4 files changed, 24 insertions(+), 29 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BeamBigQueryTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTable.java similarity index 72% rename from sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BeamBigQueryTable.java rename to sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTable.java index fdbcea4..6f3f56a 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BeamBigQueryTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTable.java @@ -20,25 +20,26 @@ package org.apache.beam.sdk.extensions.sql.meta.provider.bigquery; import java.io.Serializable; import org.apache.beam.sdk.annotations.Experimental; import org.apache.beam.sdk.extensions.sql.impl.schema.BaseBeamTable; +import org.apache.beam.sdk.extensions.sql.meta.Table; import org.apache.beam.sdk.io.gcp.bigquery.BigQueryIO; import org.apache.beam.sdk.io.gcp.bigquery.BigQueryUtils; -import org.apache.beam.sdk.schemas.Schema; +import org.apache.beam.sdk.schemas.SchemaCoder; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.POutput; import org.apache.beam.sdk.values.Row; /** - * {@code BeamBigQueryTable} represent a BigQuery table as a target. This provider does not - * currently support being a source. + * {@code BigQueryTable} represent a BigQuery table as a target. This provider does not currently + * support being a source. */ @Experimental -public class BeamBigQueryTable extends BaseBeamTable implements Serializable { - private String tableSpec; +class BigQueryTable extends BaseBeamTable implements Serializable { + String bqLocation; - public BeamBigQueryTable(Schema beamSchema, String tableSpec) { -super(beamSchema); -this.tableSpec = tableSpec; + BigQueryTable(Table table) { +super(table.getSchema()); +this.bqLocation = table.getLocation(); } @Override @@ -48,9 +49,12 @@ public class BeamBigQueryTable extends BaseBeamTable implements Serializable { @Override public PCollection buildIOReader(PBegin begin) { -// TODO: make this more generic. return begin - .apply(BigQueryIO.read(BigQueryUtils.toBeamRow(schema)).from(tableSpec)) +.apply( +"Read Input BQ Rows", +BigQueryIO.read(record -> BigQueryUtils.toBeamRow(record.getRecord(), getSchema())) +.from(bqLocation) +.withCoder(SchemaCoder.of(getSchema( .setRowSchema(getSchema()); } @@ -60,10 +64,6 @@ public class BeamBigQueryTable extends BaseBeamTable implements Serializable { BigQueryIO.write() .withSchema(BigQueryUtils.toTableSchema(getSchema())) .withFormatFunction(BigQueryUtils.toTableRow()) -.to(tableSpec)); - } - - String getTableSpec() { -return tableSpec; +.to(bqLocation)); } } diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTableProvider.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTableProvider.java index 2eceb23..e96fe29 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTableProvider.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/bigquery/BigQueryTableProvider.java @@ -22,7 +22,6 @@ import org.apache.beam.sdk.extensions.sql.BeamSqlTable; import org.apache.beam.sdk.extensions.sql.meta.Table; import org.apache.beam.sdk.extensions.sql.meta.provider.InMemoryMetaTableProvider; import org.apache.beam.sdk.extensions.sql.meta.provider.TableProvider; -import org.apache.beam.sd
[beam] branch master updated (6e315ac -> 2a15be4)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 6e315ac Merge pull request #8127 from lostluck/patch-2 new 713ee80 Added MetricsConstants class new 3ea7dd1 Utilize MonitoringInfoConstants across codebase. new 2a15be4 Merge pull request #8175 from Ardagan/CleanupTransformers The 20825 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/runners/core/metrics/MetricUrns.java | 6 +- .../core/metrics/MonitoringInfoConstants.java | 78 ++ .../core/metrics/MonitoringInfoMetricName.java | 2 +- .../runners/core/metrics/SimpleExecutionState.java | 2 +- .../core/metrics/SimpleMonitoringInfoBuilder.java | 44 ++-- .../runners/core/metrics/LabeledMetricsTest.java | 4 +- .../core/metrics/MetricsContainerImplTest.java | 6 +- .../core/metrics/MonitoringInfoMetricNameTest.java | 12 ++-- .../core/metrics/MonitoringInfoTestUtil.java | 6 +- .../core/metrics/SimpleExecutionStateTest.java | 2 +- .../metrics/SimpleMonitoringInfoBuilderTest.java | 16 ++--- .../core/metrics/SimpleStateRegistryTest.java | 14 ++-- .../metrics/SpecMonitoringInfoValidatorTest.java | 20 +++--- .../flink/metrics/FlinkMetricContainerTest.java| 14 ++-- ...ntMonitoringInfoToCounterUpdateTransformer.java | 10 +-- ...ecMonitoringInfoToCounterUpdateTransformer.java | 13 ++-- ...ntMonitoringInfoToCounterUpdateTransformer.java | 9 ++- ...onMonitoringInfoToCounterUpdateTransformer.java | 16 ++--- ...erMonitoringInfoToCounterUpdateTransformer.java | 13 ++-- .../fn/control/BeamFnMapTaskExecutorTest.java | 3 +- ...nitoringInfoToCounterUpdateTransformerTest.java | 5 +- ...nitoringInfoToCounterUpdateTransformerTest.java | 8 ++- ...nitoringInfoToCounterUpdateTransformerTest.java | 5 +- ...nitoringInfoToCounterUpdateTransformerTest.java | 5 +- ...nitoringInfoToCounterUpdateTransformerTest.java | 5 +- ...nitoringInfoToCounterUpdateTransformerTest.java | 5 +- .../fnexecution/control/RemoteExecutionTest.java | 16 +++-- .../harness/data/ElementCountFnDataReceiver.java | 7 +- .../harness/data/PCollectionConsumerRegistry.java | 6 +- .../harness/data/PTransformFunctionRegistry.java | 8 +-- .../beam/fn/harness/FnApiDoFnRunnerTest.java | 5 +- .../data/ElementCountFnDataReceiverTest.java | 3 +- 32 files changed, 215 insertions(+), 153 deletions(-) create mode 100644 runners/core-java/src/main/java/org/apache/beam/runners/core/metrics/MonitoringInfoConstants.java
[beam] branch master updated: [BEAM-6814] toListRow in BeamEnumerableConverter.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 5b2fb1c [BEAM-6814] toListRow in BeamEnumerableConverter. new 652adec Merge pull request #8044 from amaliujia/rw_add_list_row 5b2fb1c is described below commit 5b2fb1c854706369472a7afbde2e8bcabd4df659 Author: amaliujia AuthorDate: Tue Mar 12 15:51:17 2019 -0700 [BEAM-6814] toListRow in BeamEnumerableConverter. --- .../sql/impl/rel/BeamEnumerableConverter.java | 63 -- .../sql/impl/rel/BeamEnumerableConverterTest.java | 17 ++ 2 files changed, 65 insertions(+), 15 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamEnumerableConverter.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamEnumerableConverter.java index 3fb9a67..755d589 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamEnumerableConverter.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/rel/BeamEnumerableConverter.java @@ -129,6 +129,15 @@ public class BeamEnumerableConverter extends ConverterImpl implements Enumerable return options; } + static List toRowList(PipelineOptions options, BeamRelNode node) { +if (node instanceof BeamIOSinkRel) { + throw new UnsupportedOperationException("Does not support BeamIOSinkRel in toRowList."); +} else if (isLimitQuery(node)) { + throw new UnsupportedOperationException("Does not support queries with LIMIT in toRowList."); +} +return collectRowList(options, node); + } + static Enumerable toEnumerable(PipelineOptions options, BeamRelNode node) { if (node instanceof BeamIOSinkRel) { return count(options, node); @@ -143,7 +152,7 @@ public class BeamEnumerableConverter extends ConverterImpl implements Enumerable PipelineOptions options, BeamRelNode node, DoFn doFn, - Queue values, + Queue values, int limitCount) { options.as(DirectOptions.class).setBlockOnRun(false); Pipeline pipeline = Pipeline.create(options); @@ -174,9 +183,36 @@ public class BeamEnumerableConverter extends ConverterImpl implements Enumerable return result; } + private static void runCollector(PipelineOptions options, BeamRelNode node) { +Pipeline pipeline = Pipeline.create(options); +PCollection resultCollection = BeamSqlRelUtils.toPCollection(pipeline, node); +resultCollection.apply(ParDo.of(new Collector())); +PipelineResult result = pipeline.run(); +result.waitUntilFinish(); + } + + private static List collectRowList(PipelineOptions options, BeamRelNode node) { +long id = options.getOptionsId(); +Queue values = new ConcurrentLinkedQueue<>(); + +checkArgument( +options +.getRunner() +.getCanonicalName() +.equals("org.apache.beam.runners.direct.DirectRunner"), +"collectRowList is only available in direct runner."); + +Collector.globalValues.put(id, values); + +runCollector(options, node); + +Collector.globalValues.remove(id); +return values.stream().collect(Collectors.toList()); + } + private static Enumerable collect(PipelineOptions options, BeamRelNode node) { long id = options.getOptionsId(); -Queue values = new ConcurrentLinkedQueue<>(); +Queue values = new ConcurrentLinkedQueue<>(); checkArgument( options @@ -187,20 +223,16 @@ public class BeamEnumerableConverter extends ConverterImpl implements Enumerable Collector.globalValues.put(id, values); -Pipeline pipeline = Pipeline.create(options); -PCollection resultCollection = BeamSqlRelUtils.toPCollection(pipeline, node); -resultCollection.apply(ParDo.of(new Collector())); -PipelineResult result = pipeline.run(); -result.waitUntilFinish(); +runCollector(options, node); Collector.globalValues.remove(id); -return Linq4j.asEnumerable(unboxValues(values)); +return Linq4j.asEnumerable(rowToAvaticaAndUnboxValues(values)); } private static Enumerable limitCollect(PipelineOptions options, BeamRelNode node) { long id = options.getOptionsId(); -ConcurrentLinkedQueue values = new ConcurrentLinkedQueue<>(); +ConcurrentLinkedQueue values = new ConcurrentLinkedQueue<>(); checkArgument( options @@ -220,15 +252,15 @@ public class BeamEnumerableConverter extends ConverterImpl implements Enumerable values.remove(); } -return Linq4j.asEnumerable(unboxValues(values)); +return Linq4j.asEnumerable(rowToAvaticaAndUnboxValues(values)); } private static class Collector extends DoFn {
[beam] branch master updated (76ff204 -> 4c60e96)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 76ff204 [BEAM-6718] Fix BigQuery SQL postcommit (#7904) new 4093a19 [sql][hotifx] remove less useful comments new f971d78 update new 4c60e96 Merge pull request #7910 from amaliujia/rw-hot_fix The 20326 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java | 13 +++-- 1 file changed, 7 insertions(+), 6 deletions(-)
[beam] branch master updated (76ff204 -> 4c60e96)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 76ff204 [BEAM-6718] Fix BigQuery SQL postcommit (#7904) new 4093a19 [sql][hotifx] remove less useful comments new f971d78 update new 4c60e96 Merge pull request #7910 from amaliujia/rw-hot_fix The 20326 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/impl/rel/BeamCalcRel.java | 13 +++-- 1 file changed, 7 insertions(+), 6 deletions(-)
[beam] branch master updated (a921dfd -> 116600f)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from a921dfd Merge pull request #7830: Add link to contact-us page for dev@ mailing list new 13aafd1 [SQL] Hive Metastore Table provider for SQL new 039e072 [SQL] Add embedded Hive metastore test utilities new 584bcd8 Move metastore schema logic to HCatalog module new 4b84acd [SQL] Add test to join PCollection, Hive & InMemoryMetastore new d7b0054 [SQL] Make HCatalog fields nullable by default new f8864dc [SQL] Make schema in HCatToRow final new 116600f Merge pull request #7746 from akedin/hcatalog-provider The 20172 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: sdks/java/extensions/sql/build.gradle | 21 +- .../sdk/extensions/sql/impl/BeamCalciteSchema.java | 14 +- .../sql/meta/provider/TableProvider.java | 25 ++ .../meta/provider/hcatalog/DatabaseProvider.java | 100 .../sql/meta/provider/hcatalog/HCatalogTable.java | 92 +++ .../provider/hcatalog/HCatalogTableProvider.java | 110 + .../sql/meta/provider/hcatalog}/package-info.java | 4 +- .../sdk/extensions/sql/BeamSqlHiveSchemaTest.java | 268 + .../org/apache/beam/sdk/io/hcatalog/HCatToRow.java | 84 +++ .../beam/sdk/io/hcatalog/HCatalogBeamSchema.java | 86 +++ .../apache/beam/sdk/io/hcatalog/SchemaUtils.java | 66 + .../hcatalog/test}/EmbeddedMetastoreService.java | 34 ++- .../sdk/io/hcatalog/test}/HCatalogIOTestUtils.java | 34 ++- .../beam/sdk/io/hcatalog/test}/package-info.java | 6 +- .../src/{test => main}/resources/hive-site.xml | 0 .../sdk/io/hcatalog/HCatalogBeamSchemaTest.java| 104 .../apache/beam/sdk/io/hcatalog/HCatalogIOIT.java | 2 +- .../beam/sdk/io/hcatalog/HCatalogIOTest.java | 25 +- 18 files changed, 1031 insertions(+), 44 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/DatabaseProvider.java create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTable.java create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog/HCatalogTableProvider.java copy {runners/gearpump/src/main/java/org/apache/beam/runners/gearpump/translators => sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/hcatalog}/package-info.java (88%) create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamSqlHiveSchemaTest.java create mode 100644 sdks/java/io/hcatalog/src/main/java/org/apache/beam/sdk/io/hcatalog/HCatToRow.java create mode 100644 sdks/java/io/hcatalog/src/main/java/org/apache/beam/sdk/io/hcatalog/HCatalogBeamSchema.java create mode 100644 sdks/java/io/hcatalog/src/main/java/org/apache/beam/sdk/io/hcatalog/SchemaUtils.java rename sdks/java/io/hcatalog/src/{test/java/org/apache/beam/sdk/io/hcatalog => main/java/org/apache/beam/sdk/io/hcatalog/test}/EmbeddedMetastoreService.java (76%) rename sdks/java/io/hcatalog/src/{test/java/org/apache/beam/sdk/io/hcatalog => main/java/org/apache/beam/sdk/io/hcatalog/test}/HCatalogIOTestUtils.java (77%) copy sdks/java/io/{kudu/src/main/java/org/apache/beam/sdk/io/kudu => hcatalog/src/main/java/org/apache/beam/sdk/io/hcatalog/test}/package-info.java (85%) rename sdks/java/io/hcatalog/src/{test => main}/resources/hive-site.xml (100%) create mode 100644 sdks/java/io/hcatalog/src/test/java/org/apache/beam/sdk/io/hcatalog/HCatalogBeamSchemaTest.java
[beam] branch master updated: [BEAM-6574] Inline description of CSV formats
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new b4d1846 [BEAM-6574] Inline description of CSV formats new af6b869 Merge pull request #7697 from kennknowles/CSVFormat b4d1846 is described below commit b4d1846ad134484be0e50e2851e100303690ce60 Author: Kenneth Knowles AuthorDate: Thu Jan 31 16:49:19 2019 -0800 [BEAM-6574] Inline description of CSV formats --- .../src/documentation/dsls/sql/create-external-table.md | 15 +-- 1 file changed, 13 insertions(+), 2 deletions(-) diff --git a/website/src/documentation/dsls/sql/create-external-table.md b/website/src/documentation/dsls/sql/create-external-table.md index a6f6e32..f332880 100644 --- a/website/src/documentation/dsls/sql/create-external-table.md +++ b/website/src/documentation/dsls/sql/create-external-table.md @@ -323,8 +323,19 @@ TBLPROPERTIES '{"format: "Excel"}' * `LOCATION`: The path to the file for Read Mode. The prefix for Write Mode. * `TBLPROPERTIES`: -* `format`: Optional. Allows you to specify the - [CSVFormat](https://commons.apache.org/proper/commons-csv/archives/1.5/apidocs/org/apache/commons/csv/CSVFormat.Predefined.html). +* `format`: Optional. Allows you to specify the CSV Format, which controls +the field delimeter, quote character, record separator, and other properties. +See the following table: + + +| Value for `format` | Field delimiter | Quote | Record separator | Ignore empty lines? | Allow missing column names? | +||-|---|--|-|-| +| `default` | `,` | `"` | `\r\n` | Yes | No | +| `rfc4180` | `,` | `"` | `\r\n` | No | No | +| `excel`| `,` | `"` | `\r\n` | No | Yes | +| `tdf` | `\t`| `"` | `\r\n` | Yes | No | +| `mysql`| `\t`| none | `\n` | No | No | +{:.table-bordered} ### Read Mode
[beam] branch master updated: [SQL] Fix CREATE EXTERNAL TABLE in non-default schema (#7627)
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 5b46b02 [SQL] Fix CREATE EXTERNAL TABLE in non-default schema (#7627) 5b46b02 is described below commit 5b46b02b49ca1c5c18682427a5a4a25596ca4287 Author: Gleb Kanterov AuthorDate: Sat Jan 26 18:08:37 2019 +0100 [SQL] Fix CREATE EXTERNAL TABLE in non-default schema (#7627) * [SQL] Fix CREATE EXTERNAL TABLE in non-default schema --- .../sql/impl/parser/SqlCreateExternalTable.java| 2 +- .../extensions/sql/impl/parser/SqlDdlNodes.java| 13 -- .../sdk/extensions/sql/impl/BeamSqlEnvTest.java| 49 ++ .../extensions/sql/impl/parser/BeamDDLTest.java| 14 +++ 4 files changed, 73 insertions(+), 5 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlCreateExternalTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlCreateExternalTable.java index 1ff2276..7041e33 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlCreateExternalTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlCreateExternalTable.java @@ -151,7 +151,7 @@ public class SqlCreateExternalTable extends SqlCreate implements SqlExecutableSt private Table toTable() { return Table.builder() .type(SqlDdlNodes.getString(type)) -.name(name.getSimple()) +.name(SqlDdlNodes.name(name)) .schema(columnList.stream().collect(toSchema())) .comment(SqlDdlNodes.getString(comment)) .location(SqlDdlNodes.getString(location)) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlDdlNodes.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlDdlNodes.java index c508502..d9ceeb5 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlDdlNodes.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/parser/SqlDdlNodes.java @@ -48,20 +48,25 @@ public class SqlDdlNodes { /** Returns the schema in which to create an object. */ static Pair schema( CalcitePrepare.Context context, boolean mutable, SqlIdentifier id) { -final String name; final List path; if (id.isSimple()) { path = context.getDefaultSchemaPath(); - name = id.getSimple(); } else { path = Util.skipLast(id.names); - name = Util.last(id.names); } CalciteSchema schema = mutable ? context.getMutableRootSchema() : context.getRootSchema(); for (String p : path) { schema = schema.getSubSchema(p, true); } -return Pair.of(schema, name); +return Pair.of(schema, name(id)); + } + + static String name(SqlIdentifier id) { +if (id.isSimple()) { + return id.getSimple(); +} else { + return Util.last(id.names); +} } static @Nullable String getString(SqlNode n) { diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnvTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnvTest.java new file mode 100644 index 000..b812e4c --- /dev/null +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnvTest.java @@ -0,0 +1,49 @@ +/* + * 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.beam.sdk.extensions.sql.impl; + +import static org.junit.Assert.assertEquals; + +import java.sql.Connection; +import java.sql.ResultSet; +import org.apache.beam.sdk.extensions.sql.meta.provider.test.TestTableProvider; +import org.junit.Test; + +/** Tests for {@link BeamSqlEnv}. */ +public class BeamSqlEnvTest { + + @Test + public void testCreateExternalTableInNestedTableProvider() throws Exception { +TestTableProvider root = new TestTableProvider(); +Test
[beam] branch master updated: Fix couple of design-docs links for SQL
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 12e46b1 Fix couple of design-docs links for SQL new f560edc Merge pull request #7630 from akedin/fix-links 12e46b1 is described below commit 12e46b1a969483ee2e31dfd79bf379323b028a13 Author: akedin AuthorDate: Fri Jan 25 13:03:22 2019 -0800 Fix couple of design-docs links for SQL --- website/src/contribute/design-documents.md | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) diff --git a/website/src/contribute/design-documents.md b/website/src/contribute/design-documents.md index 6472bba..1cc0c81 100644 --- a/website/src/contribute/design-documents.md +++ b/website/src/contribute/design-documents.md @@ -83,10 +83,10 @@ This is a collection of documents that may or may not be up to date. - Streams and Tables [[doc](https://s.apache.org/beam-streams-tables)] - Streaming SQL [[doc](http://s.apache.org/streaming-sql-spec)] - Schema-Aware PCollections [[doc](https://docs.google.com/document/d/1tnG2DPHZYbsomvihIpXruUmQ12pHGK0QIvXS1FOTgRc)] -- Pubsub to Beam SQL [[doc](https://docs.google.com/document/d/1wIXTxh-nQ3u694XbF0iEZX_7-b3yi4ad0ML2pcAxYfE)] +- Pubsub to Beam SQL [[doc](https://docs.google.com/document/d/1554kJD33ovkBDvSNjasHu90L_EZOS26ZHr4ao1muS-A)] - Apache Beam Proposal: design of DSL SQL interface [[doc](https://docs.google.com/document/d/1uWXL_yF3UUO5GfCxbL6kWsmC8xCWfICU3RwiQKsk7Mk)] -- Calcite/Beam SQL Windowing [[doc](https://docs.google.com/document/d/1RmyV9e1Qab-axsLI1WWpw5oGAJDv0X7y9OSnPnrZWJk)] -- Reject Unsupported Windowing Strategies in JOIN [[doc](https://docs.google.com/document/d/1V-ZgKVTwHdNSGlQWncWIzcf_Rw2oLKZFSkU43scLff4)] +- Calcite/Beam SQL Windowing [[doc](https://docs.google.com/document/d/1yuG_fAnbAKEq3qz2jdf8qxyEIZ3xJAbCF1bbd_Y9Ia8)] +- Reject Unsupported Windowing Strategies in JOIN [[doc](https://docs.google.com/document/d/1Me0orPfH6vEFjfsTGcZ5ELWg-sw4st1ZvXqYyr7Pexc)] - Beam DSL_SQL branch API review [[doc](https://s.apache.org/beam-sql-dsl-api-review)] - Complex Types Support for Beam SQL DDL [[mail](https://lists.apache.org/thread.html/c494e521cb6865b1ae19a68e8e653afc562df7744e8d08087249cbe0@%3Cdev.beam.apache.org%3E)] - [SQL] Reject unsupported inputs to Joins [[mail](https://lists.apache.org/thread.html/e7a442fa9cf6b76a5b435493170508f6c42fb9ccef9bcef434424f79@%3Cdev.beam.apache.org%3E)]
[beam] branch master updated: [BEAM-6468] Allow creating empty TestBoundedTable.
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 4142f55 [BEAM-6468] Allow creating empty TestBoundedTable. new 50d0760 Merge pull request #7568 from amaliujia/rui_wang-fix_testbounded_table 4142f55 is described below commit 4142f5509ca0fce2e57b8b1c3420bad58baa8611 Author: amaliujia AuthorDate: Fri Jan 18 12:15:20 2019 -0800 [BEAM-6468] Allow creating empty TestBoundedTable. --- .../sql/meta/provider/test/TestBoundedTable.java | 4 +- .../sdk/extensions/sql/TestBoundedTableTest.java | 43 ++ 2 files changed, 46 insertions(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/test/TestBoundedTable.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/test/TestBoundedTable.java index 9a64791..5c92c47 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/test/TestBoundedTable.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/test/TestBoundedTable.java @@ -93,7 +93,9 @@ public class TestBoundedTable extends TestTable { @Override public PCollection buildIOReader(PBegin begin) { return begin -.apply("MockedBoundedTable_Reader_" + COUNTER.incrementAndGet(), Create.of(rows)) +.apply( +"MockedBoundedTable_Reader_" + COUNTER.incrementAndGet(), +Create.of(rows).withRowSchema(schema)) .setRowSchema(getSchema()); } diff --git a/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/TestBoundedTableTest.java b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/TestBoundedTableTest.java new file mode 100644 index 000..2657b30 --- /dev/null +++ b/sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/TestBoundedTableTest.java @@ -0,0 +1,43 @@ +/* + * 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.beam.sdk.extensions.sql; + +import org.apache.beam.sdk.extensions.sql.meta.provider.test.TestBoundedTable; +import org.apache.beam.sdk.schemas.Schema; +import org.apache.beam.sdk.testing.TestPipeline; +import org.apache.beam.sdk.values.PBegin; +import org.junit.Rule; +import org.junit.Test; +import org.junit.runner.RunWith; +import org.junit.runners.JUnit4; + +/** test TestBoundedTable class. */ +@RunWith(JUnit4.class) +public class TestBoundedTableTest { + + @Rule public final TestPipeline pipeline = TestPipeline.create(); + + @Test + public void testCreatingEmptyTable() { +TestBoundedTable emptyTable = +TestBoundedTable.of( + Schema.builder().addInt32Field("ColId").addStringField("Value").build()); +emptyTable.buildIOReader(PBegin.in(pipeline)); +pipeline.run(); + } +}
[beam] branch master updated: make getCurrentSchemaPlus public
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new e5e66bf make getCurrentSchemaPlus public new b1603ce Merge pull request #7609 from amaliujia/rui_wang-make_getschema_public e5e66bf is described below commit e5e66bfa3967049d43853bc29005f6197a5d3324 Author: amaliujia AuthorDate: Wed Jan 23 13:25:17 2019 -0800 make getCurrentSchemaPlus public --- .../java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java| 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java index e3c2aa6..1955ec6 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java @@ -108,7 +108,7 @@ public class JdbcConnection extends CalciteConnectionWrapper { } /** Calcite-created {@link SchemaPlus} wrapper for the current schema. */ - SchemaPlus getCurrentSchemaPlus() { + public SchemaPlus getCurrentSchemaPlus() { try { return getRootSchema().getSubSchema(getSchema()); } catch (SQLException e) {
[beam] branch master updated: [SQL] Force cacheless root schema in Jdbc connection
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 20d6093 [SQL] Force cacheless root schema in Jdbc connection new 6a59667 Merge pull request #7553 from akedin/cacheless-root-schema 20d6093 is described below commit 20d6093e4a4f864091d821d5320a5468a7547b4f Author: akedin AuthorDate: Thu Jan 17 16:04:34 2019 -0800 [SQL] Force cacheless root schema in Jdbc connection --- .../extensions/sql/impl/CalciteFactoryWrapper.java | 110 .../sdk/extensions/sql/impl/JdbcConnection.java| 26 ++--- .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 68 ++-- .../beam/sdk/extensions/sql/impl/JdbcFactory.java | 115 + .../sdk/extensions/sql/impl/JdbcDriverTest.java| 12 +++ 5 files changed, 262 insertions(+), 69 deletions(-) diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteFactoryWrapper.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteFactoryWrapper.java new file mode 100644 index 000..a039154 --- /dev/null +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteFactoryWrapper.java @@ -0,0 +1,110 @@ +/* + * 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.beam.sdk.extensions.sql.impl; + +import java.sql.ResultSetMetaData; +import java.sql.SQLException; +import java.util.Properties; +import java.util.TimeZone; +import org.apache.calcite.adapter.java.JavaTypeFactory; +import org.apache.calcite.avatica.AvaticaConnection; +import org.apache.calcite.avatica.AvaticaFactory; +import org.apache.calcite.avatica.AvaticaPreparedStatement; +import org.apache.calcite.avatica.AvaticaResultSet; +import org.apache.calcite.avatica.AvaticaSpecificDatabaseMetaData; +import org.apache.calcite.avatica.AvaticaStatement; +import org.apache.calcite.avatica.Meta; +import org.apache.calcite.avatica.QueryState; +import org.apache.calcite.avatica.UnregisteredDriver; +import org.apache.calcite.jdbc.CalciteFactory; +import org.apache.calcite.jdbc.CalciteSchema; + +/** + * Wrapper for {@link CalciteFactory}. + * + * This is a non-functional class to delegate to the underlying {@link CalciteFactory}. The + * purpose is to hide the delegation logic from the implementation ({@link JdbcFactory}). + */ +public abstract class CalciteFactoryWrapper extends CalciteFactory { + + protected CalciteFactory factory; + + CalciteFactoryWrapper(CalciteFactory factory) { +super(4, 1); +this.factory = factory; + } + + @Override + public AvaticaConnection newConnection( + UnregisteredDriver driver, + AvaticaFactory avaticaFactory, + String url, + Properties info, + CalciteSchema rootSchema, + JavaTypeFactory typeFactory) { + +return this.factory.newConnection(driver, avaticaFactory, url, info, rootSchema, typeFactory); + } + + @Override + public AvaticaStatement newStatement( + AvaticaConnection connection, + Meta.StatementHandle h, + int resultSetType, + int resultSetConcurrency, + int resultSetHoldability) + throws SQLException { +return this.factory.newStatement( +connection, h, resultSetType, resultSetConcurrency, resultSetHoldability); + } + + @Override + public AvaticaPreparedStatement newPreparedStatement( + AvaticaConnection connection, + Meta.StatementHandle h, + Meta.Signature signature, + int resultSetType, + int resultSetConcurrency, + int resultSetHoldability) + throws SQLException { +return this.factory.newPreparedStatement( +connection, h, signature, resultSetType, resultSetConcurrency, resultSetHoldability); + } + + @Override + public AvaticaResultSet newResultSet( + AvaticaStatement statement, + QueryState state, + Meta.Signature signature, + TimeZone timeZone, + Meta.Frame firstFrame) + throws SQLException { +return this.factory.newResultSet(statement, state, signature, t
[beam] 01/01: Merge pull request #7491 from akedin/multiple-schemas-support
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit feece918d2a938efd3fed22e7ce4e127953f662d Merge: ff70ca1 de08064 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Tue Jan 15 12:11:19 2019 -0800 Merge pull request #7491 from akedin/multiple-schemas-support [SQL] Multiple schemas support .../beam/sdk/extensions/sql/SqlTransform.java | 26 +++ .../sdk/extensions/sql/impl/BeamQueryPlanner.java | 29 +-- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 13 ++ .../sdk/extensions/sql/impl/JdbcConnection.java| 1 + .../extensions/sql/BeamSqlMultipleSchemasTest.java | 253 + 5 files changed, 309 insertions(+), 13 deletions(-)
[beam] branch master updated (ff70ca1 -> feece91)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from ff70ca1 Moving to 2.11.0-SNAPSHOT on master branch add 9ff8001 [SQL] Support multiple top-level schemas in SqlTransform add de08064 [SQL] Support changing the default schema in SqlTransform new feece91 Merge pull request #7491 from akedin/multiple-schemas-support The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/sdk/extensions/sql/SqlTransform.java | 26 +++ .../sdk/extensions/sql/impl/BeamQueryPlanner.java | 29 +-- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 13 ++ .../sdk/extensions/sql/impl/JdbcConnection.java| 1 + .../extensions/sql/BeamSqlMultipleSchemasTest.java | 253 + 5 files changed, 309 insertions(+), 13 deletions(-) create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamSqlMultipleSchemasTest.java
[beam] branch master updated (c38d84c -> f8a5dd6)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from c38d84c Merge pull request #7284: [BEAM-6234] Add failOnCheckpointingErrors setting in FlinkPipelineOptions add 73cb1b9 [SQL] Refactor JdbcDriver add f8a5dd6 Merge pull request #7488 from akedin/refactor-jdbc-driver No new revisions were added by this update. Summary of changes: .../sdk/extensions/sql/impl/BeamCalciteSchema.java | 28 +- .../sql/impl/BeamCalciteSchemaFactory.java | 180 +- .../sdk/extensions/sql/impl/BeamQueryPlanner.java | 5 +- .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 15 +- .../sql/impl/CalciteConnectionWrapper.java | 400 + .../sdk/extensions/sql/impl/JdbcConnection.java| 128 +++ .../beam/sdk/extensions/sql/impl/JdbcDriver.java | 149 .../sql/impl/parser/SqlSetOptionBeam.java | 21 +- .../sql/meta/provider/TableProvider.java | 5 + .../sql/meta/provider/pubsub/PubsubJsonIT.java | 17 +- 10 files changed, 820 insertions(+), 128 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/CalciteConnectionWrapper.java create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/JdbcConnection.java
[beam] branch master updated (bac909b -> f190152)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from bac909b Treat VarInt encoding as a Beam primitive encoding in Dataflow runner (#7351) add f720985 [BEAM-6110] For SQL CoGBK-based join use EARLIEST output timestamp new f190152 Merge pull request #7115 from kennknowles/sql-join-cogbk-timestamps The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java | 8 1 file changed, 8 insertions(+)
[beam] 01/01: Merge pull request #7115 from kennknowles/sql-join-cogbk-timestamps
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit f1901527238ff3d6b1b505696a91bbd34827a0c8 Merge: bac909b f720985 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Thu Dec 27 09:11:25 2018 -0800 Merge pull request #7115 from kennknowles/sql-join-cogbk-timestamps [BEAM-6110] For SQL CoGBK-based join use EARLIEST output timestamp .../org/apache/beam/sdk/extensions/sql/impl/rel/BeamJoinRel.java | 8 1 file changed, 8 insertions(+)
[beam] branch master updated (772fa67 -> 2f4214e)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 772fa67 [BEAM-5320] Finish Python 3 porting for testing module (#7029) add ab55b3f [BEAM-5852][SQL] Disallow checked exceptions in UDF new 2f4214e Merge pull request #7096 from apilloud/exception The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../java/org/apache/beam/sdk/extensions/sql/impl/UdfImpl.java | 3 +++ .../sdk/extensions/sql/impl/udf/BuiltinStringFunctions.java| 10 +++--- 2 files changed, 10 insertions(+), 3 deletions(-)
[beam] 01/01: Merge pull request #7096 from apilloud/exception
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 2f4214ec2e51633438d249d882e0212461853a17 Merge: 772fa67 ab55b3f Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Tue Nov 20 17:46:44 2018 -0500 Merge pull request #7096 from apilloud/exception [BEAM-5852][SQL] Disallow checked exceptions in UDF .../java/org/apache/beam/sdk/extensions/sql/impl/UdfImpl.java | 3 +++ .../sdk/extensions/sql/impl/udf/BuiltinStringFunctions.java| 10 +++--- 2 files changed, 10 insertions(+), 3 deletions(-)
[beam] branch master updated (68d72e5 -> 1cbbc4d)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 68d72e5 Merge pull request #7044: [BEAM-5456] Upgrades Google API Client libraries to 1.27.0. add 3008f5a [BEAM-6083] use same parsing functions to parse time add 1cbbc4d Merge pull request #7072 from amaliujia/rw_parse_time No new revisions were added by this update. Summary of changes: .../beam/sdk/extensions/sql/BeamSqlCliTest.java| 6 +-- .../extensions/sql/BeamSqlDslAggregationTest.java | 49 +-- .../beam/sdk/extensions/sql/BeamSqlDslBase.java| 13 ++--- .../sdk/extensions/sql/BeamSqlDslJoinTest.java | 3 +- .../sql/BeamSqlDslSqlStdOperatorsTest.java | 54 +++-- .../sdk/extensions/sql/BeamSqlDslUdfUdafTest.java | 9 +++- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 29 +++ .../provider/bigquery/BigQueryReadWriteIT.java | 7 ++- .../provider/pubsub/PubsubMessageToRowTest.java| 4 +- .../sdk/extensions/sql/utils/DateTimeUtils.java| 56 ++ 10 files changed, 137 insertions(+), 93 deletions(-) create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/utils/DateTimeUtils.java
[beam] branch master updated (4b38e63 -> dc01eca)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 4b38e63 Merge pull request #6685: [BEAM-5744] Add unit tests that uncover the regression observed in PR #6602 add d9fe1b0 [SQL] Add extra cast tests for string->date parsing add dc01eca Merge pull request #6743 from akedin/date-cast No new revisions were added by this update. Summary of changes: .../beam/sdk/extensions/sql/BeamSqlCastTest.java | 137 + 1 file changed, 137 insertions(+) create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/BeamSqlCastTest.java
[beam] branch master updated (7f48e5a -> f65b485)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7f48e5a [BEAM-6013] Reduce logging within SerializableCoder. (#6982) add 71860a6 [BEAM-5943][BEAM-5940] 1. Add ENDS_WITH. 2. Add LEGNTH. add 92d561e [BEAM-5945] add reverse add 7abdfa9 [BEAM-5952] add STARTS_WITH. add b95558b [SQL] add NULL check for builtin functions. add aebb7b7 [BEAM-5852] add FROM_HEX add adc1fed [BEAM-5962] Add TO_HEX add 91f9a4b [BEAM-5963]: add LPAD add 139f4d1 [BEAM-5965] Add RPAD new f65b485 Merge pull request #6928 from amaliujia/rui_wang-add_string_functions The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../interpreter/operator/BeamSqlPrimitive.java | 1 + .../sql/impl/udf/BuiltinStringFunctions.java | 302 + .../impl/udf/BuiltinTrigonometricFunctions.java| 9 + .../extensions/sql/impl/utils/CalciteUtils.java| 3 + .../beam/sdk/extensions/sql/BeamSqlDslBase.java| 84 ++ .../udf/BeamSalUhfSpecialTypeAndValueTest.java | 170 .../sql/impl/udf/BeamSqlUdfExpressionTest.java | 202 ++ .../sql/impl/udf/BeamSqlUdfImplTest.java | 139 -- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 + 9 files changed, 778 insertions(+), 139 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/BuiltinStringFunctions.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/udf/BeamSalUhfSpecialTypeAndValueTest.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/udf/BeamSqlUdfExpressionTest.java delete mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/udf/BeamSqlUdfImplTest.java
[beam] 01/01: Merge pull request #6928 from amaliujia/rui_wang-add_string_functions
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit f65b4850502137b274c4e73c91cacf38da48 Merge: 7f48e5a 139f4d1 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Thu Nov 8 17:01:26 2018 -0500 Merge pull request #6928 from amaliujia/rui_wang-add_string_functions [BEAM-5852] BeamSQL functions .../interpreter/operator/BeamSqlPrimitive.java | 1 + .../sql/impl/udf/BuiltinStringFunctions.java | 302 + .../impl/udf/BuiltinTrigonometricFunctions.java| 9 + .../extensions/sql/impl/utils/CalciteUtils.java| 3 + .../beam/sdk/extensions/sql/BeamSqlDslBase.java| 84 ++ .../udf/BeamSalUhfSpecialTypeAndValueTest.java | 170 .../sql/impl/udf/BeamSqlUdfExpressionTest.java | 202 ++ .../sql/impl/udf/BeamSqlUdfImplTest.java | 139 -- ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 7 + 9 files changed, 778 insertions(+), 139 deletions(-) diff --cc sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/CalciteUtils.java index cd93166,f5c89a2..5c22314 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/CalciteUtils.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/utils/CalciteUtils.java @@@ -202,8 -203,10 +203,10 @@@ public class CalciteUtils */ public static RelDataType sqlTypeWithAutoCast(RelDataTypeFactory typeFactory, Type rawType) { //For Joda time types, return SQL type for java.util.Date. -if (rawType instanceof Class && ReadableInstant.class.isAssignableFrom((Class) rawType)) { +if (rawType instanceof Class && AbstractInstant.class.isAssignableFrom((Class) rawType)) { return typeFactory.createJavaType(Date.class); + } else if (rawType instanceof Class && ByteString.class.isAssignableFrom((Class) rawType)) { + return typeFactory.createJavaType(byte[].class); } return typeFactory.createJavaType((Class) rawType); }
[beam] branch master updated (e1a2a88 -> df432d3)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e1a2a88 Merge pull request #6975 from ruoyu90/textsink add ecc5882 [BEAM-5979] support DATE and TIME in DML new df432d3 Merge pull request #6967 from amaliujia/rui_wang-support_date_time The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../extensions/sql/impl/schema/BeamTableUtils.java | 3 +- .../sql/meta/provider/test/TestTableProvider.java | 7 +++- .../beam/sdk/extensions/sql/BeamSqlCliTest.java| 41 ++ 3 files changed, 49 insertions(+), 2 deletions(-)
[beam] 01/01: Merge pull request #6967 from amaliujia/rui_wang-support_date_time
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit df432d3c8ef109767ea26ca255f1473fcfd1 Merge: e1a2a88 ecc5882 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Wed Nov 7 13:31:20 2018 -0500 Merge pull request #6967 from amaliujia/rui_wang-support_date_time [BEAM-5979] Fix DATE and TIME in INSERTION .../extensions/sql/impl/schema/BeamTableUtils.java | 3 +- .../sql/meta/provider/test/TestTableProvider.java | 7 +++- .../beam/sdk/extensions/sql/BeamSqlCliTest.java| 41 ++ 3 files changed, 49 insertions(+), 2 deletions(-)
[beam] branch master updated (ba5bc60 -> 5fd8257)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from ba5bc60 Merge pull request #6964: [BEAM-6010] Deprecate KafkaIO withTimestampFn(). add 73af056 [BEAM-6006] Test TIME +/- INTERVAL new 5fd8257 Merge pull request #6972 from amaliujia/rui_wang-test_time_add_sub The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../date/BeamSqlDatetimeMinusIntervalExpression.java | 4 +++- .../sdk/extensions/sql/BeamSqlDslSqlStdOperatorsTest.java | 14 ++ .../date/BeamSqlDatetimeMinusIntervalExpressionTest.java | 1 + .../BeamSqlBuiltinFunctionsIntegrationTestBase.java| 9 + 4 files changed, 23 insertions(+), 5 deletions(-)
[beam] 01/01: Merge pull request #6972 from amaliujia/rui_wang-test_time_add_sub
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 5fd82572b5a6ae84f6313f86a98a6230e8104a36 Merge: ba5bc60 73af056 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Wed Nov 7 12:53:44 2018 -0500 Merge pull request #6972 from amaliujia/rui_wang-test_time_add_sub [BEAM-6006] Test TIME +/- INTERVAL .../date/BeamSqlDatetimeMinusIntervalExpression.java | 4 +++- .../sdk/extensions/sql/BeamSqlDslSqlStdOperatorsTest.java | 14 ++ .../date/BeamSqlDatetimeMinusIntervalExpressionTest.java | 1 + .../BeamSqlBuiltinFunctionsIntegrationTestBase.java| 9 + 4 files changed, 23 insertions(+), 5 deletions(-)
[beam] 01/01: Merge pull request #6968 from amaliujia/rui_wang-date_add_sub
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 3bde7db6ba7c6b9e75239fc43514e17183e923c6 Merge: 41bf693 96f9115 Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Tue Nov 6 19:32:20 2018 -0500 Merge pull request #6968 from amaliujia/rui_wang-date_add_sub [BEAM-6001] Fix DATE - INTERVAL and test DATE +/- INTERVAL. .../sql/impl/interpreter/BeamSqlFnExecutor.java| 4 +- .../BeamSqlDatetimeMinusIntervalExpression.java| 2 +- .../sql/BeamSqlDslSqlStdOperatorsTest.java | 58 -- ...BeamSqlDatetimeMinusIntervalExpressionTest.java | 1 + ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 16 +++--- 5 files changed, 46 insertions(+), 35 deletions(-)
[beam] branch master updated (41bf693 -> 3bde7db)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 41bf693 [BEAM-5915] Timeout http requests, if they got stuck, at dataflow job creation time (#6922) add 96f9115 [BEAM-6001] fix DATE - INTERVAL, test DATE +/- INTERVAL. new 3bde7db Merge pull request #6968 from amaliujia/rui_wang-date_add_sub The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../sql/impl/interpreter/BeamSqlFnExecutor.java| 4 +- .../BeamSqlDatetimeMinusIntervalExpression.java| 2 +- .../sql/BeamSqlDslSqlStdOperatorsTest.java | 58 -- ...BeamSqlDatetimeMinusIntervalExpressionTest.java | 1 + ...BeamSqlBuiltinFunctionsIntegrationTestBase.java | 16 +++--- 5 files changed, 46 insertions(+), 35 deletions(-)
[beam] branch master updated (8a57b47 -> e6fdb70)
This is an automated email from the ASF dual-hosted git repository. anton pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 8a57b47 Merge pull request #6896: Add separate pre-commit job for spotless validation add 8d35bf195 [BEAM-5852] add IS_INF and IS_NAN. add afe9f07 [BEAM-5885] support COSH add 1f8258b [BEAM-5889] support SINH add b5cb338 [BEAM-5890] add TANH add 1a0b96a [SQL] spotless apply add 9fdca2f [BEAM-5892] add annotation to help register built-in functions. add 723c17b [SQL] improve beam builtin function registration. add e6fdb70 Merge pull request #6898 from amaliujia/rui_wang-add_some_udfs No new revisions were added by this update. Summary of changes: .../beam/sdk/extensions/sql/SqlTransform.java | 2 + .../beam/sdk/extensions/sql/impl/BeamSqlEnv.java | 18 +++ .../sql/impl/udf/BeamBuiltinFunctionProvider.java | 23 +++- .../impl/udf/BuiltinTrigonometricFunctions.java| 70 +++ .../beam/sdk/extensions/sql/impl/udf/IsInf.java} | 38 -- .../beam/sdk/extensions/sql/impl/udf/IsNan.java} | 37 -- .../beam/sdk/extensions/sql/impl/udf/UDF.java} | 21 ++-- .../sdk/extensions/sql/impl/udf}/package-info.java | 4 +- .../beam/sdk/extensions/sql/BeamSqlDslBase.java| 43 ++- .../sql/impl/udf/BeamSqlUdfImplTest.java | 139 + 10 files changed, 352 insertions(+), 43 deletions(-) copy runners/samza/src/main/java/org/apache/beam/runners/samza/translation/SamzaTranslatorRegistrar.java => sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/BeamBuiltinFunctionProvider.java (57%) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/BuiltinTrigonometricFunctions.java copy sdks/java/{core/src/main/java/org/apache/beam/sdk/options/PipelineOptionsRegistrar.java => extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/IsInf.java} (50%) copy sdks/java/{core/src/main/java/org/apache/beam/sdk/options/PipelineOptionsRegistrar.java => extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/IsNan.java} (51%) copy sdks/java/{core/src/main/java/org/apache/beam/sdk/util/SystemDoFnInternal.java => extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf/UDF.java} (76%) copy sdks/java/{testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/queries => extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/udf}/package-info.java (91%) create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/impl/udf/BeamSqlUdfImplTest.java
[beam] 01/01: Merge pull request #6856 from reuvenlax/sql_use_group
This is an automated email from the ASF dual-hosted git repository. anton pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit d8a5faa4fe91d910517fb091fade0a2ad4b40c5f Merge: 401d663 be6942e Author: Anton Kedin <33067037+ake...@users.noreply.github.com> AuthorDate: Sat Oct 27 01:40:09 2018 -0400 Merge pull request #6856 from reuvenlax/sql_use_group [BEAM-4461] Switch SQL over to use the new Beam Group transform. .../apache/beam/sdk/schemas/transforms/Group.java | 142 +++ .../sql/impl/rel/BeamAggregationRel.java | 129 +++--- .../sql/impl/transform/MultipleAggregationsFn.java | 175 --- .../impl/transform/agg/AggregationArgsAdapter.java | 171 -- .../transform/agg/AggregationCombineFnAdapter.java | 193 - 5 files changed, 308 insertions(+), 502 deletions(-)