Spark Streaming + Kafka Integration Guide

Apache Kafka is publish-subscribe messaging rethought as a distributed, partitioned, replicated commit log service. Here we explain how to configure Spark Streaming to receive data from Kafka. There are two approaches to this - the old approach using Receivers and Kafka’s high-level API, and a new experimental approach (introduced in Spark 1.3) without using Receivers. They have different programming models, performance characteristics, and semantics guarantees, so read on for more details.

Approach 1: Receiver-based Approach

This approach uses a Receiver to receive the data. The Received is implemented using the Kafka high-level consumer API. As with all receivers, the data received from Kafka through a Receiver is stored in Spark executors, and then jobs launched by Spark Streaming processes the data.

However, under default configuration, this approach can lose data under failures (see receiver reliability. To ensure zero-data loss, you have to additionally enable Write Ahead Logs in Spark Streaming. To ensure zero data loss, enable the Write Ahead Logs (introduced in Spark 1.2). This synchronously saves all the received Kafka data into write ahead logs on a distributed file system (e.g HDFS), so that all the data can be recovered on failure. See Deploying section in the streaming programming guide for more details on Write Ahead Logs.

Next, we discuss how to use this approach in your streaming application.

  1. Linking: For Scala/Java applications using SBT/Maven project definitions, link your streaming application with the following artifact (see Linking section in the main programming guide for further information).

     groupId = org.apache.spark
     artifactId = spark-streaming-kafka_2.10
     version = 1.3.0
    

    For Python applications, you will have to add this above library and its dependencies when deploying your application. See the Deploying subsection below.

  2. Programming: In the streaming application code, import KafkaUtils and create an input DStream as follows.

     import org.apache.spark.streaming.kafka._
    
     val kafkaStream = KafkaUtils.createStream(streamingContext, 
         [ZK quorum], [consumer group id], [per-topic number of Kafka partitions to consume])
    

    You can also specify the key and value classes and their corresponding decoder classes using variations of createStream. See the API docs and the example.

     import org.apache.spark.streaming.kafka.*;
    
     JavaPairReceiverInputDStream<String, String> kafkaStream = 
         KafkaUtils.createStream(streamingContext,
         [ZK quorum], [consumer group id], [per-topic number of Kafka partitions to consume]);
    

    You can also specify the key and value classes and their corresponding decoder classes using variations of createStream. See the API docs and the example.

     from pyspark.streaming.kafka import KafkaUtils
    
     kafkaStream = KafkaUtils.createStream(streamingContext, \
         [ZK quorum], [consumer group id], [per-topic number of Kafka partitions to consume])
    

    By default, the Python API will decode Kafka data as UTF8 encoded strings. You can specify your custom decoding function to decode the byte arrays in Kafka records to any arbitrary data type. See the API docs and the example.

    Points to remember:

    • Topic partitions in Kafka does not correlate to partitions of RDDs generated in Spark Streaming. So increasing the number of topic-specific partitions in the KafkaUtils.createStream() only increases the number of threads using which topics that are consumed within a single receiver. It does not increase the parallelism of Spark in processing the data. Refer to the main document for more information on that.

    • Multiple Kafka input DStreams can be created with different groups and topics for parallel receiving of data using multiple receivers.

    • If you have enabled Write Ahead Logs with a replicated file system like HDFS, the received data is already being replicated in the log. Hence, the storage level in storage level for the input stream to StorageLevel.MEMORY_AND_DISK_SER (that is, use KafkaUtils.createStream(..., StorageLevel.MEMORY_AND_DISK_SER)).

  3. Deploying: As with any Spark applications, spark-submit is used to launch your application. However, the details are slightly different for Scala/Java applications and Python applications.

    For Scala and Java applications, if you are using SBT or Maven for project management, then package spark-streaming-kafka_2.10 and its dependencies into the application JAR. Make sure spark-core_2.10 and spark-streaming_2.10 are marked as provided dependencies as those are already present in a Spark installation. Then use spark-submit to launch your application (see Deploying section in the main programming guide).

    For Python applications which lack SBT/Maven project management, spark-streaming-kafka_2.10 and its dependencies can be directly added to spark-submit using --packages (see Application Submission Guide). That is,

     ./bin/spark-submit --packages org.apache.spark:spark-streaming-kafka_2.10:1.3.0 ...
    

    Alternatively, you can also download the JAR of the Maven artifact spark-streaming-kafka-assembly from the Maven repository and add it to spark-submit with --jars.

Approach 2: Direct Approach (No Receivers)

This is a new receiver-less “direct” approach has been introduced in Spark 1.3 to ensure stronger end-to-end guarantees. Instead of using receivers to receive data, this approach periodically queries Kafka for the latest offsets in each topic+partition, and accordingly defines the offset ranges to process in each batch. When the jobs to process the data are launched, Kafka’s simple consumer API is used to read the defined ranges of offsets from Kafka (similar to read files from a file system). Note that this is an experimental feature in Spark 1.3 and is only available in the Scala and Java API.

This approach has the following advantages over the received-based approach (i.e. Approach 1).

Note that one disadvantage of this approach is that it does not update offsets in Zookeeper, hence Zookeeper-based Kafka monitoring tools will not show progress. However, you can access the offsets processed by this approach in each batch and update Zookeeper yourself (see below).

Next, we discuss how to use this approach in your streaming application.

  1. Linking: This approach is supported only in Scala/Java application. Link your SBT/Maven project with the following artifact (see Linking section in the main programming guide for further information).

     groupId = org.apache.spark
     artifactId = spark-streaming-kafka_2.10
     version = 1.3.0
    
  2. Programming: In the streaming application code, import KafkaUtils and create an input DStream as follows.

     import org.apache.spark.streaming.kafka._
    
     val directKafkaStream = KafkaUtils.createDirectStream[
         [key class], [value class], [key decoder class], [value decoder class] ](
         streamingContext, [map of Kafka parameters], [set of topics to consume])
    

    See the API docs and the example.

     import org.apache.spark.streaming.kafka.*;
    
     JavaPairReceiverInputDStream<String, String> directKafkaStream = 
         KafkaUtils.createDirectStream(streamingContext,
             [key class], [value class], [key decoder class], [value decoder class],
             [map of Kafka parameters], [set of topics to consume]);
    

    See the API docs and the example.

    In the Kafka parameters, you must specify either metadata.broker.list or bootstrap.servers. By default, it will start consuming from the latest offset of each Kafka partition. If you set configuration auto.offset.reset in Kafka parameters to smallest, then it will start consuming from the smallest offset.

    You can also start consuming from any arbitrary offset using other variations of KafkaUtils.createDirectStream. Furthermore, if you want to access the Kafka offsets consumed in each batch, you can do the following.

     directKafkaStream.foreachRDD { rdd => 
         val offsetRanges = rdd.asInstanceOf[HasOffsetRanges]
         // offsetRanges.length = # of Kafka partitions being consumed
         ...
     }
    
     directKafkaStream.foreachRDD(
         new Function<JavaPairRDD<String, String>, Void>() {
             @Override
             public Void call(JavaPairRDD<String, Integer> rdd) throws IOException {
                 OffsetRange[] offsetRanges = ((HasOffsetRanges)rdd).offsetRanges
                 // offsetRanges.length = # of Kafka partitions being consumed
                 ...
                 return null;
             }
         }
     );
    

    You can use this to update Zookeeper yourself if you want Zookeeper-based Kafka monitoring tools to show progress of the streaming application.

    Another thing to note is that since this approach does not use Receivers, the standard receiver-related (that is, configurations of the form spark.streaming.receiver.* ) will not apply to the input DStreams created by this approach (will apply to other input DStreams though). Instead, use the configurations spark.streaming.kafka.*. An important one is spark.streaming.kafka.maxRatePerPartition which is the maximum rate at which each Kafka partition will be read by this direct API.

  3. Deploying: Similar to the first approach, you can package spark-streaming-kafka_2.10 and its dependencies into the application JAR and the launch the application using spark-submit. Make sure spark-core_2.10 and spark-streaming_2.10 are marked as provided dependencies as those are already present in a Spark installation.