You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by sn...@apache.org on 2022/10/12 20:52:06 UTC

[beam] branch master updated: use write schema only for read api (#23594)

This is an automated email from the ASF dual-hosted git repository.

sniemitz 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 8e52d8721b7 use write schema only for read api (#23594)
8e52d8721b7 is described below

commit 8e52d8721b7c9e5408096a68892d312067eb63e8
Author: Kanishk Karanawat <kk...@gmail.com>
AuthorDate: Wed Oct 12 16:51:56 2022 -0400

    use write schema only for read api (#23594)
    
    Co-authored-by: Kanishk Karanawat <kk...@twitter.com>
---
 .../org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java   | 15 ++++-----------
 .../beam/sdk/io/gcp/bigquery/BigQueryIOReadTest.java      |  2 +-
 2 files changed, 5 insertions(+), 12 deletions(-)

diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java
index 760609988d5..e72bc20f780 100644
--- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java
+++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java
@@ -595,20 +595,17 @@ public class BigQueryIO {
     private final Supplier<TableSchema> tableSchema;
     private GenericDatumReader<T> reader;
     private org.apache.avro.Schema writerSchema;
-    private org.apache.avro.Schema readerSchema;
 
     public GenericDatumTransformer(
         SerializableFunction<SchemaAndRecord, T> parseFn,
         String tableSchema,
-        org.apache.avro.Schema writer,
-        org.apache.avro.Schema reader) {
+        org.apache.avro.Schema writer) {
       this.parseFn = parseFn;
       this.tableSchema =
           Suppliers.memoize(
               Suppliers.compose(new TableSchemaFunction(), Suppliers.ofInstance(tableSchema)));
       this.writerSchema = writer;
-      this.readerSchema = reader;
-      this.reader = new GenericDatumReader<>(this.writerSchema, this.readerSchema);
+      this.reader = new GenericDatumReader<>(this.writerSchema);
     }
 
     @Override
@@ -618,11 +615,7 @@ public class BigQueryIO {
       }
 
       this.writerSchema = schema;
-      if (this.readerSchema == null) {
-        this.readerSchema = schema;
-      }
-
-      this.reader = new GenericDatumReader<>(this.writerSchema, this.readerSchema);
+      this.reader = new GenericDatumReader<>(this.writerSchema);
     }
 
     @Override
@@ -664,7 +657,7 @@ public class BigQueryIO {
                     String jsonTableSchema = BigQueryIO.JSON_FACTORY.toString(input);
                     return (AvroSource.DatumReaderFactory<T>)
                         (writer, reader) ->
-                            new GenericDatumTransformer<>(parseFn, jsonTableSchema, writer, reader);
+                            new GenericDatumTransformer<>(parseFn, jsonTableSchema, writer);
                   } catch (IOException e) {
                     LOG.warn(
                         String.format("Error while converting table schema %s to JSON!", input), e);
diff --git a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIOReadTest.java b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIOReadTest.java
index 75202b4076b..8fbee3af002 100644
--- a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIOReadTest.java
+++ b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIOReadTest.java
@@ -143,7 +143,7 @@ public class BigQueryIOReadTest implements Serializable {
                   return (AvroSource.DatumReaderFactory<TableRow>)
                       (writer, reader) ->
                           new BigQueryIO.GenericDatumTransformer<>(
-                              BigQueryIO.TableRowParser.INSTANCE, jsonSchema, writer, reader);
+                              BigQueryIO.TableRowParser.INSTANCE, jsonSchema, writer);
                 } catch (IOException e) {
                   return null;
                 }