Class Trigger

Object
org.apache.spark.sql.streaming.Trigger

@Evolving public class Trigger extends Object
Policy used to indicate how often results should be produced by a [[StreamingQuery]].
Since:
2.0.0
  • Constructor Summary

    Constructors
    Constructor
    Description
     
  • Method Summary

    Modifier and Type
    Method
    Description
    static Trigger
    A trigger that processes all available data at the start of the query in one or multiple batches, then terminates the query.
    static Trigger
    Continuous(long intervalMs)
    A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval.
    static Trigger
    Continuous(long interval, TimeUnit timeUnit)
    A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval.
    static Trigger
    Continuous(String interval)
    A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval.
    static Trigger
    Continuous(scala.concurrent.duration.Duration interval)
    (Scala-friendly) A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval.
    static Trigger
    Deprecated.
    This is deprecated as of Spark 3.4.0.
    static Trigger
    ProcessingTime(long intervalMs)
    A trigger policy that runs a query periodically based on an interval in processing time.
    static Trigger
    ProcessingTime(long interval, TimeUnit timeUnit)
    (Java-friendly) A trigger policy that runs a query periodically based on an interval in processing time.
    static Trigger
    A trigger policy that runs a query periodically based on an interval in processing time.
    static Trigger
    ProcessingTime(scala.concurrent.duration.Duration interval)
    (Scala-friendly) A trigger policy that runs a query periodically based on an interval in processing time.

    Methods inherited from class java.lang.Object

    equals, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
  • Constructor Details

    • Trigger

      public Trigger()
  • Method Details

    • ProcessingTime

      public static Trigger ProcessingTime(long intervalMs)
      A trigger policy that runs a query periodically based on an interval in processing time. If `interval` is 0, the query will run as fast as possible.
      Since:
      2.2.0
    • ProcessingTime

      public static Trigger ProcessingTime(long interval, TimeUnit timeUnit)
      (Java-friendly) A trigger policy that runs a query periodically based on an interval in processing time. If `interval` is 0, the query will run as fast as possible. {{{ import java.util.concurrent.TimeUnit df.writeStream().trigger(Trigger.ProcessingTime(10, TimeUnit.SECONDS)) }}}
      Since:
      2.2.0
    • ProcessingTime

      public static Trigger ProcessingTime(scala.concurrent.duration.Duration interval)
      (Scala-friendly) A trigger policy that runs a query periodically based on an interval in processing time. If `duration` is 0, the query will run as fast as possible. {{{ import scala.concurrent.duration._ df.writeStream.trigger(Trigger.ProcessingTime(10.seconds)) }}}
      Since:
      2.2.0
    • ProcessingTime

      public static Trigger ProcessingTime(String interval)
      A trigger policy that runs a query periodically based on an interval in processing time. If `interval` is effectively 0, the query will run as fast as possible. {{{ df.writeStream.trigger(Trigger.ProcessingTime("10 seconds")) }}}
      Since:
      2.2.0
    • Once

      @Deprecated(since="3.4.0") public static Trigger Once()
      Deprecated.
      This is deprecated as of Spark 3.4.0. Use AvailableNow() to leverage better guarantee of processing, fine-grained scale of batches, and better gradual processing of watermark advancement including no-data batch. See the NOTES in AvailableNow() for details.
      A trigger that processes all available data in a single batch then terminates the query.
      Since:
      2.2.0
    • AvailableNow

      public static Trigger AvailableNow()
      A trigger that processes all available data at the start of the query in one or multiple batches, then terminates the query. Users are encouraged to set the source options to control the size of the batch as similar as controlling the size of the batch in ProcessingTime(long) trigger. NOTES: - This trigger provides a strong guarantee of processing: regardless of how many batches were left over in previous run, it ensures all available data at the time of execution gets processed before termination. All uncommitted batches will be processed first. - Watermark gets advanced per each batch, and no-data batch gets executed before termination if the last batch advances the watermark. This helps to maintain smaller and predictable state size and smaller latency on the output of stateful operators.
      Since:
      3.3.0
    • Continuous

      public static Trigger Continuous(long intervalMs)
      A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval.
      Since:
      2.3.0
    • Continuous

      public static Trigger Continuous(long interval, TimeUnit timeUnit)
      A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval. {{{ import java.util.concurrent.TimeUnit df.writeStream.trigger(Trigger.Continuous(10, TimeUnit.SECONDS)) }}}
      Since:
      2.3.0
    • Continuous

      public static Trigger Continuous(scala.concurrent.duration.Duration interval)
      (Scala-friendly) A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval. {{{ import scala.concurrent.duration._ df.writeStream.trigger(Trigger.Continuous(10.seconds)) }}}
      Since:
      2.3.0
    • Continuous

      public static Trigger Continuous(String interval)
      A trigger that continuously processes streaming data, asynchronously checkpointing at the specified interval. {{{ df.writeStream.trigger(Trigger.Continuous("10 seconds")) }}}
      Since:
      2.3.0