Uploaded image for project: 'Beam'
  1. Beam
  2. BEAM-12256

PubsubIO.readAvroGenericRecord creates SchemaCoder that fails to decode some Avro logical types

Details

    • Bug
    • Status: Open
    • P1
    • Resolution: Unresolved
    • 2.29.0
    • None
    • io-java-gcp, sdk-java-core
    • None

    Description

      For example, when PubsubIO.readAvroGenericRecord is used with an avro schema that includes a decimal type with a non-zero scale, it can encounter the following exception when decoding:

      org.apache.avro.AvroTypeException: Cannot encode decimal with scale 17 as scale 0
      	at org.apache.avro.Conversions$DecimalConversion.toBytes(Conversions.java:92)
      	at org.apache.beam.sdk.schemas.utils.AvroUtils.genericFromBeamField(AvroUtils.java:975)
      	at org.apache.beam.sdk.schemas.utils.AvroUtils.toGenericRecord(AvroUtils.java:397)
      	at org.apache.beam.sdk.schemas.utils.AvroUtils$RowToGenericRecordFn.apply(AvroUtils.java:547)
      	at org.apache.beam.sdk.schemas.utils.AvroUtils$RowToGenericRecordFn.apply(AvroUtils.java:538)
      	at org.apache.beam.sdk.schemas.SchemaCoder.decode(SchemaCoder.java:123)
      	at org.apache.beam.sdk.io.gcp.bigquery.TableRowInfoCoder.decode(TableRowInfoCoder.java:64)
      	at org.apache.beam.sdk.io.gcp.bigquery.TableRowInfoCoder.decode(TableRowInfoCoder.java:30)
      	at org.apache.beam.runners.dataflow.worker.WindmillKeyedWorkItem.lambda$elementsIterable$2(WindmillKeyedWorkItem.java:112)
      	at org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.Iterators$6.transform(Iterators.java:785)
      	at org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.TransformedIterator.next(TransformedIterator.java:47)
      	at org.apache.beam.runners.dataflow.worker.StreamingGroupAlsoByWindowReshuffleFn.processElement(StreamingGroupAlsoByWindowReshuffleFn.java:56)
      	at org.apache.beam.runners.dataflow.worker.StreamingGroupAlsoByWindowReshuffleFn.processElement(StreamingGroupAlsoByWindowReshuffleFn.java:39)
      	at org.apache.beam.runners.dataflow.worker.GroupAlsoByWindowFnRunner.invokeProcessElement(GroupAlsoByWindowFnRunner.java:121)
      	at org.apache.beam.runners.dataflow.worker.GroupAlsoByWindowFnRunner.processElement(GroupAlsoByWindowFnRunner.java:73)
      	at org.apache.beam.runners.dataflow.worker.GroupAlsoByWindowsParDoFn.processElement(GroupAlsoByWindowsParDoFn.java:137)
      	at org.apache.beam.runners.dataflow.worker.util.common.worker.ParDoOperation.process(ParDoOperation.java:44)
      	at org.apache.beam.runners.dataflow.worker.util.common.worker.OutputReceiver.process(OutputReceiver.java:49)
      	at org.apache.beam.runners.dataflow.worker.util.common.worker.ReadOperation.runReadLoop(ReadOperation.java:212)
      	at org.apache.beam.runners.dataflow.worker.util.common.worker.ReadOperation.start(ReadOperation.java:163)
      	at org.apache.beam.runners.dataflow.worker.util.common.worker.MapTaskExecutor.execute(MapTaskExecutor.java:92)
      	at org.apache.beam.runners.dataflow.worker.StreamingDataflowWorker.process(StreamingDataflowWorker.java:1426)
      	at org.apache.beam.runners.dataflow.worker.StreamingDataflowWorker.access$1100(StreamingDataflowWorker.java:163)
      	at org.apache.beam.runners.dataflow.worker.StreamingDataflowWorker$7.run(StreamingDataflowWorker.java:1105)
      	at java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
      	at java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
      	at java.lang.Thread.run(Thread.java:748) 

      Attachments

        Issue Links

          Activity

            People

              Unassigned Unassigned
              bhulette Brian Hulette
              Votes:
              0 Vote for this issue
              Watchers:
              5 Start watching this issue

              Dates

                Created:
                Updated:

                Time Tracking

                  Estimated:
                  Original Estimate - Not Specified
                  Not Specified
                  Remaining:
                  Remaining Estimate - 0h
                  0h
                  Logged:
                  Time Spent - 2h
                  2h