aboutsummaryrefslogtreecommitdiff
path: root/mllib
diff options
context:
space:
mode:
authorMechCoder <manojkumarsivaraj334@gmail.com>2015-06-19 12:23:15 -0700
committerXiangrui Meng <meng@databricks.com>2015-06-19 12:23:15 -0700
commit54976e55e36465108b71b40b8a431be9d6d703ce (patch)
treecacad3ffa4f48e89ce575684272456e7b3931937 /mllib
parente41e2fd6c61076f870de03b85c5da6c12b8da038 (diff)
downloadspark-54976e55e36465108b71b40b8a431be9d6d703ce.tar.gz
spark-54976e55e36465108b71b40b8a431be9d6d703ce.tar.bz2
spark-54976e55e36465108b71b40b8a431be9d6d703ce.zip
[SPARK-4118] [MLLIB] [PYSPARK] Python bindings for StreamingKMeans
Python bindings for StreamingKMeans Will change status to MRG once docs, tests and examples are updated. Author: MechCoder <manojkumarsivaraj334@gmail.com> Closes #6499 from MechCoder/spark-4118 and squashes the following commits: 7722d16 [MechCoder] minor style fixes 51052d3 [MechCoder] Doc fixes 2061a76 [MechCoder] Add tests for simultaneous training and prediction Minor style fixes 81482fd [MechCoder] minor 5d9fe61 [MechCoder] predictOn should take into account the latest model 8ab9e89 [MechCoder] Fix Python3 error a9817df [MechCoder] Better tests and minor fixes c80e451 [MechCoder] Add ignore_unicode_prefix ee8ce16 [MechCoder] Update tests, doc and examples 4b1481f [MechCoder] Some changes and tests d8b066a [MechCoder] [SPARK-4118] [MLlib] [PySpark] Python bindings for StreamingKMeans
Diffstat (limited to 'mllib')
-rw-r--r--mllib/src/main/scala/org/apache/spark/mllib/api/python/PythonMLLibAPI.scala15
1 files changed, 15 insertions, 0 deletions
diff --git a/mllib/src/main/scala/org/apache/spark/mllib/api/python/PythonMLLibAPI.scala b/mllib/src/main/scala/org/apache/spark/mllib/api/python/PythonMLLibAPI.scala
index 1812b3ac7c..2897865af6 100644
--- a/mllib/src/main/scala/org/apache/spark/mllib/api/python/PythonMLLibAPI.scala
+++ b/mllib/src/main/scala/org/apache/spark/mllib/api/python/PythonMLLibAPI.scala
@@ -964,6 +964,21 @@ private[python] class PythonMLLibAPI extends Serializable {
points.asScala.toArray)
}
+ /**
+ * Java stub for the update method of StreamingKMeansModel.
+ */
+ def updateStreamingKMeansModel(
+ clusterCenters: JList[Vector],
+ clusterWeights: JList[Double],
+ data: JavaRDD[Vector],
+ decayFactor: Double,
+ timeUnit: String): JList[Object] = {
+ val model = new StreamingKMeansModel(
+ clusterCenters.asScala.toArray, clusterWeights.asScala.toArray)
+ .update(data, decayFactor, timeUnit)
+ List[AnyRef](model.clusterCenters, Vectors.dense(model.clusterWeights)).asJava
+ }
+
}
/**