Class BigtableIO.ReadChangeStream

  • All Implemented Interfaces:
    java.io.Serializable, org.apache.beam.sdk.transforms.display.HasDisplayData
    Enclosing class:
    BigtableIO

    public abstract static class BigtableIO.ReadChangeStream
    extends org.apache.beam.sdk.transforms.PTransform<org.apache.beam.sdk.values.PBegin,​org.apache.beam.sdk.values.PCollection<org.apache.beam.sdk.values.KV<com.google.protobuf.ByteString,​com.google.cloud.bigtable.data.v2.models.ChangeStreamMutation>>>
    See Also:
    Serialized Form
    • Constructor Detail

      • ReadChangeStream

        public ReadChangeStream()
    • Method Detail

      • withMetadataTableProjectId

        public BigtableIO.ReadChangeStream withMetadataTableProjectId​(java.lang.String projectId)
        Returns a new BigtableIO.ReadChangeStream that will use the Cloud Bigtable project indicated by given parameter to manage the metadata of the stream.

        Optional: defaults to value from withProjectId

        Does not modify this object.

      • withMetadataTableInstanceId

        public BigtableIO.ReadChangeStream withMetadataTableInstanceId​(java.lang.String instanceId)
        Returns a new BigtableIO.ReadChangeStream that will use the Cloud Bigtable instance indicated by given parameter to manage the metadata of the stream.

        Optional: defaults to value from withInstanceId

        Does not modify this object.

      • withMetadataTableTableId

        public BigtableIO.ReadChangeStream withMetadataTableTableId​(java.lang.String tableId)
        Returns a new BigtableIO.ReadChangeStream that will use specified table to store the metadata of the stream.

        Optional: defaults to value from withTableId

        Does not modify this object.

      • withMetadataTableAppProfileId

        public BigtableIO.ReadChangeStream withMetadataTableAppProfileId​(java.lang.String appProfileId)
        Returns a new BigtableIO.ReadChangeStream that will use the cluster specified by app profile id to store the metadata of the stream.

        Optional: defaults to value from withAppProfileId

        This must use single-cluster routing policy with allow single-row transactions enabled.

        Does not modify this object.

      • withCreateOrUpdateMetadataTable

        public BigtableIO.ReadChangeStream withCreateOrUpdateMetadataTable​(boolean shouldCreate)
        Returns a new BigtableIO.ReadChangeStream that, if set to true, will create or update metadata table before launching pipeline. Otherwise, it is expected that a metadata table with correct schema exists.

        Optional: defaults to true

        Does not modify this object.

      • withBacklogReplicationAdjustment

        public BigtableIO.ReadChangeStream withBacklogReplicationAdjustment​(org.joda.time.Duration adjustment)
        Returns a new BigtableIO.ReadChangeStream that overrides the replication delay adjustment duration with the provided duration.

        Backlog is calculated for each partition using watermarkLag * throughput. Replication delay holds back the watermark for each partition. This can cause the backlog to stay persistently above dataflow's downscaling threshold (10 seconds) even when a pipeline is caught up.

        This adjusts the backlog downward to account for this. For unreplicated instances it can be set to zero to upscale as quickly as possible.

        Optional: defaults to 30 seconds.

        Does not modify this object.

      • withReadChangeStreamTimeout

        public BigtableIO.ReadChangeStream withReadChangeStreamTimeout​(org.joda.time.Duration timeout)
        Returns a new BigtableIO.ReadChangeStream that overrides timeout for ReadChangeStream requests.

        This is useful to override the default of 15s timeout if the checkpoint duration is longer than 15s. Setting this value to longer (to add some padding) than periodic checkpoint duration ensures that ReadChangeStream will stream until the next checkpoint is initiated.

        Optional: defaults to 15 seconds.

        Does not modify this object.

      • withoutValidation

        public BigtableIO.ReadChangeStream withoutValidation()
        Disables validation that the table being read and the metadata table exists, and that the app profile used is single cluster and single row transaction enabled. Set this option if the caller does not have additional Bigtable permissions to validate the configurations. NOTE this also disabled creating or updating the metadata table because that also requires additional permissions, essentially setting withCreateOrUpdateMetadataTable(boolean) to false.
      • validate

        public void validate​(org.apache.beam.sdk.options.PipelineOptions options)
        Overrides:
        validate in class org.apache.beam.sdk.transforms.PTransform<org.apache.beam.sdk.values.PBegin,​org.apache.beam.sdk.values.PCollection<org.apache.beam.sdk.values.KV<com.google.protobuf.ByteString,​com.google.cloud.bigtable.data.v2.models.ChangeStreamMutation>>>
      • expand

        public org.apache.beam.sdk.values.PCollection<org.apache.beam.sdk.values.KV<com.google.protobuf.ByteString,​com.google.cloud.bigtable.data.v2.models.ChangeStreamMutation>> expand​(org.apache.beam.sdk.values.PBegin input)
        Specified by:
        expand in class org.apache.beam.sdk.transforms.PTransform<org.apache.beam.sdk.values.PBegin,​org.apache.beam.sdk.values.PCollection<org.apache.beam.sdk.values.KV<com.google.protobuf.ByteString,​com.google.cloud.bigtable.data.v2.models.ChangeStreamMutation>>>