Skip to content

Commit 75d485e

Browse files
author
Davies Liu
committed
add mqtt
1 parent 07923c4 commit 75d485e

File tree

2 files changed

+54
-1
lines changed

2 files changed

+54
-1
lines changed

python/pyspark/streaming/kafka.py

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -68,4 +68,4 @@ def createStream(ssc, zkQuorum, groupId, topics,
6868
stream = stream.map(lambda (k, v): (keyDecoder(k), v))
6969
if valueDecoder is not None:
7070
stream = stream.mapValues(valueDecoder)
71-
return stream
71+
return stream

python/pyspark/streaming/mqtt.py

Lines changed: 53 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,53 @@
1+
#
2+
# Licensed to the Apache Software Foundation (ASF) under one or more
3+
# contributor license agreements. See the NOTICE file distributed with
4+
# this work for additional information regarding copyright ownership.
5+
# The ASF licenses this file to You under the Apache License, Version 2.0
6+
# (the "License"); you may not use this file except in compliance with
7+
# the License. You may obtain a copy of the License at
8+
#
9+
# http://www.apache.org/licenses/LICENSE-2.0
10+
#
11+
# Unless required by applicable law or agreed to in writing, software
12+
# distributed under the License is distributed on an "AS IS" BASIS,
13+
# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
# See the License for the specific language governing permissions and
15+
# limitations under the License.
16+
#
17+
18+
19+
from py4j.java_gateway import java_import, Py4JError
20+
21+
from pyspark.storagelevel import StorageLevel
22+
from pyspark.serializers import UTF8Deserializer
23+
from pyspark.streaming import DStream
24+
25+
__all__ = ['MQTTUtils']
26+
27+
28+
class MQTTUtils(object):
29+
30+
@staticmethod
31+
def createStream(ssc, brokerUrl, topic, storageLevel=StorageLevel.MEMORY_AND_DISK_SER_2):
32+
"""
33+
Create an input stream that receives messages pushed by a MQTT publisher.
34+
35+
:param ssc: StreamingContext object
36+
:param brokerUrl: Url of remote MQTT publisher
37+
:param topic: Topic name to subscribe to
38+
:param storageLevel: RDD storage level.
39+
:return: A DStream object
40+
"""
41+
java_import(ssc._jvm, "org.apache.spark.streaming.mqtt.MQTTUtils")
42+
jlevel = ssc._sc._getJavaStorageLevel(storageLevel)
43+
try:
44+
jstream = ssc._jvm.MQTTUtils.createStream(ssc._jssc, brokerUrl, topic, jlevel)
45+
except Py4JError, e:
46+
if 'call a package' in e.message:
47+
print "No MQTT package, please build it and add it into classpath:"
48+
print " $ sbt/sbt streaming-mqtt/package"
49+
print " $ bin/submit --driver-class-path external/mqtt/target/scala-2.10/" \
50+
"spark-streaming-mqtt_2.10-1.3.0-SNAPSHOT.jar"
51+
raise Exception("No mqtt package")
52+
raise e
53+
return DStream(jstream, ssc, UTF8Deserializer())

0 commit comments

Comments
 (0)