[beam] branch master updated: [SQL] Make BigQuery schema conversion order-aware

2019-04-10 Thread anton
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.sdk.schemas.Schema;
 
 /**

[beam] branch master updated: [BEAM-7024] Calcite BINARY to Beam Schema BYTES missing in CalciteUtils

2019-04-10 Thread kenn
This is an automated email from the ASF dual-hosted git repository.

kenn 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 2e41bbb  [BEAM-7024] Calcite BINARY to Beam Schema BYTES missing in 
CalciteUtils
 new 3c74212  Merge pull request #8242: [BEAM-7024] Calcite BINARY to Beam 
Schema BYTES missing in CalciteUtils
2e41bbb is described below

commit 2e41bbb1133e7c55a0289cc3d00a427668d4fa7b
Author: amaliujia 
AuthorDate: Fri Apr 5 15:40:07 2019 -0700

[BEAM-7024] Calcite BINARY to Beam Schema BYTES missing in CalciteUtils
---
 .../extensions/sql/impl/utils/CalciteUtils.java| 22 --
 1 file changed, 20 insertions(+), 2 deletions(-)

diff --git 
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
index 02eb1b5..207900f 100644
--- 
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
@@ -156,8 +156,26 @@ public class CalciteUtils {
   .put(TIMESTAMP_WITH_LOCAL_TZ, 
SqlTypeName.TIMESTAMP_WITH_LOCAL_TIME_ZONE)
   .build();
 
-  private static final BiMap 
CALCITE_TO_BEAM_TYPE_MAPPING =
-  BEAM_TO_CALCITE_TYPE_MAPPING.inverse();
+  private static final ImmutableMap 
CALCITE_TO_BEAM_TYPE_MAPPING =
+  ImmutableMap.builder()
+  .put(SqlTypeName.TINYINT, TINY_INT)
+  .put(SqlTypeName.SMALLINT, SMALL_INT)
+  .put(SqlTypeName.INTEGER, INTEGER)
+  .put(SqlTypeName.BIGINT, BIG_INT)
+  .put(SqlTypeName.FLOAT, FLOAT)
+  .put(SqlTypeName.DOUBLE, DOUBLE)
+  .put(SqlTypeName.DECIMAL, DECIMAL)
+  .put(SqlTypeName.BOOLEAN, BOOLEAN)
+  .put(SqlTypeName.VARBINARY, VARBINARY)
+  .put(SqlTypeName.BINARY, VARBINARY)
+  .put(SqlTypeName.VARCHAR, VARCHAR)
+  .put(SqlTypeName.CHAR, CHAR)
+  .put(SqlTypeName.DATE, DATE)
+  .put(SqlTypeName.TIME, TIME)
+  .put(SqlTypeName.TIME_WITH_LOCAL_TIME_ZONE, TIME_WITH_LOCAL_TZ)
+  .put(SqlTypeName.TIMESTAMP, TIMESTAMP)
+  .put(SqlTypeName.TIMESTAMP_WITH_LOCAL_TIME_ZONE, 
TIMESTAMP_WITH_LOCAL_TZ)
+  .build();
 
   // Since there are multiple Calcite type that correspond to a single Beam 
type, this is the
   // default mapping.



[beam] branch master updated: [BEAM-7042] remove antlr from shadow configuration

2019-04-10 Thread iemejia
This is an automated email from the ASF dual-hosted git repository.

iemejia 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 18d0464  [BEAM-7042] remove antlr from shadow configuration
 new cc698bc  Merge pull request #8268: [BEAM-7042] remove antlr from 
shadow configuration
18d0464 is described below

commit 18d0464dcc2382a95cefd97527c949d2775ccd35
Author: Michael Luckey <25622840+adude3...@users.noreply.github.com>
AuthorDate: Wed Apr 10 13:34:51 2019 +0200

[BEAM-7042] remove antlr from shadow configuration
---
 sdks/java/core/build.gradle | 1 -
 1 file changed, 1 deletion(-)

diff --git a/sdks/java/core/build.gradle b/sdks/java/core/build.gradle
index a4a1264..f36144c 100644
--- a/sdks/java/core/build.gradle
+++ b/sdks/java/core/build.gradle
@@ -69,7 +69,6 @@ dependencies {
   compile library.java.commons_compress
   compile library.java.commons_lang3
   compile library.java.guava_testlib
-  shadow library.java.antlr
   shadow library.java.jackson_core
   shadow library.java.jackson_annotations
   shadow library.java.jackson_databind



[beam] branch master updated (c8a8c12 -> 59dc930)

2019-04-10 Thread lgajowy
This is an automated email from the ASF dual-hosted git repository.

lgajowy pushed a change to branch master
in repository https://gitbox.apache.org/repos/asf/beam.git.


from c8a8c12  SamzaRunner: Remove the LinkedIn repo accidentally committed 
(#8259)
 new 9103f3a  BEAM-6627 Move IOITMetrics to a common test package, move BQ 
options to common IOIT option class
 new 679f742  [BEAM-6627] Add metric reporting to JdbcIOIT
 new b130ae5  [BEAM-6627] Add metrics gathering and reporting to 
MongoDBIOIT. Modify Jenkins test job
 new 0465641  [BEAM-6627] Add metrics gathering and reporting to 
HadoopInputFormatIOIT. Modify Jenkins test job accordingly
 new 59dc930  Merge pull request #8159: [Beam-6627] Metric gathering and 
reporting in JDBC, MongoDB and HadoopInputFormat IOITs.

The 20920 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:
 .../job_PerformanceTests_HadoopInputFormat.groovy  |  4 +-
 .../jenkins/job_PerformanceTests_JDBC.groovy   |  4 +-
 .../job_PerformanceTests_MongoDBIO_IT.groovy   |  4 +-
 .../beam/sdk/io/common/IOTestPipelineOptions.java  | 13 
 .../java/org/apache/beam/sdk/io/avro/AvroIOIT.java |  2 +-
 .../io/common/FileBasedIOTestPipelineOptions.java  | 12 ---
 .../apache/beam/sdk/io/parquet/ParquetIOIT.java|  2 +-
 .../java/org/apache/beam/sdk/io/text/TextIOIT.java |  2 +-
 .../apache/beam/sdk/io/tfrecord/TFRecordIOIT.java  |  2 +-
 .../java/org/apache/beam/sdk/io/xml/XmlIOIT.java   |  2 +-
 sdks/java/io/hadoop-input-format/build.gradle  |  1 +
 .../hadoop/inputformat/HadoopInputFormatIOIT.java  | 67 -
 sdks/java/io/jdbc/build.gradle |  1 +
 .../java/org/apache/beam/sdk/io/jdbc/JdbcIOIT.java | 86 +++---
 sdks/java/io/mongodb/build.gradle  |  1 +
 .../apache/beam/sdk/io/mongodb/MongoDBIOIT.java| 66 -
 .../beam/sdk/testutils/metrics}/IOITMetrics.java   |  3 +-
 17 files changed, 234 insertions(+), 38 deletions(-)
 rename 
sdks/java/{io/file-based-io-tests/src/test/java/org/apache/beam/sdk/io/common 
=> 
testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics}/IOITMetrics.java
 (95%)