@@ -20,43 +20,44 @@ package org.apache.spark.graphx.lib
2020import scala .reflect .ClassTag
2121import org .apache .spark .graphx ._
2222
23- /** LPA algorithm. */
24- object LPA {
23+ /** Label Propagation algorithm. */
24+ object LabelPropagation {
2525 /**
26- * Run LPA (label propogation algorithm) for detecting communities in networks using the pregel framework.
27- *
28- * Each node in the network is initially assigned to its own community. At every super step
29- * nodes send their community affiliation to all neighbors and update their state to the mode
30- * community affiliation of incomming messages.
26+ * Run static Label Propagation for detecting communities in networks.
3127 *
32- * LPA is a standard community detection algorithm for graphs. It is very inexpensive
28+ * Each node in the network is initially assigned to its own community. At every superstep, nodes
29+ * send their community affiliation to all neighbors and update their state to the mode community
30+ * affiliation of incoming messages.
31+ *
32+ * LPA is a standard community detection algorithm for graphs. It is very inexpensive
3333 * computationally, although (1) convergence is not guaranteed and (2) one can end up with
3434 * trivial solutions (all nodes are identified into a single community).
3535 *
36- * @tparam VD the vertex attribute type (discarded in the computation)
3736 * @tparam ED the edge attribute type (not used in the computation)
3837 *
3938 * @param graph the graph for which to compute the community affiliation
40- * @param maxSteps the number of supersteps of LPA to be performed
39+ * @param maxSteps the number of supersteps of LPA to be performed. Because this is a static
40+ * implementation, the algorithm will run for exactly this many supersteps.
4141 *
4242 * @return a graph with vertex attributes containing the label of community affiliation
4343 */
44- def run [VD : ClassTag , ED : ClassTag ](graph : Graph [VD , ED ], maxSteps : Int ): Graph [VertexId , Long ] {
44+ def run [ED : ClassTag ](graph : Graph [_ , ED ], maxSteps : Int ): Graph [VertexId , ED ] = {
4545 val lpaGraph = graph.mapVertices { case (vid, _) => vid }
46- def sendMessage (edge : EdgeTriplet [VertexId , ED ]) = {
47- Iterator ((e.srcId, Map (e.dstAttr -> 1L )),(e.dstId, Map (e.srcAttr -> 1L )))
46+ def sendMessage (e : EdgeTriplet [VertexId , ED ]) = {
47+ Iterator ((e.srcId, Map (e.dstAttr -> 1L )), (e.dstId, Map (e.srcAttr -> 1L )))
4848 }
49- def mergeMessage (count1 : Map [VertexId , Long ], count2 : Map [VertexId , Long ]): Map [VertexId , Long ] = {
49+ def mergeMessage (count1 : Map [VertexId , Long ], count2 : Map [VertexId , Long ])
50+ : Map [VertexId , Long ] = {
5051 (count1.keySet ++ count2.keySet).map { i =>
51- val count1Val = count1.getOrElse(i,0L )
52- val count2Val = count2.getOrElse(i,0L )
53- i -> (count1Val + count2Val)
54- }.toMap
52+ val count1Val = count1.getOrElse(i, 0L )
53+ val count2Val = count2.getOrElse(i, 0L )
54+ i -> (count1Val + count2Val)
55+ }.toMap
5556 }
56- def vertexProgram (vid : VertexId , attr : Long , message : Map [VertexId , Long ])= {
57- if (message.isEmpty) attr else message.maxBy{ _._2} ._1),
57+ def vertexProgram (vid : VertexId , attr : Long , message : Map [VertexId , Long ]) = {
58+ if (message.isEmpty) attr else message.maxBy( _._2) ._1
5859 }
59- val initialMessage = Map [VertexId ,Long ]()
60+ val initialMessage = Map [VertexId , Long ]()
6061 Pregel (lpaGraph, initialMessage, maxIterations = maxSteps)(
6162 vprog = vertexProgram,
6263 sendMsg = sendMessage,
0 commit comments