2019-09-10 02:14:24 +00:00
|
|
|
## Nim-LibP2P
|
2019-09-24 17:48:23 +00:00
|
|
|
## Copyright (c) 2019 Status Research & Development GmbH
|
2019-09-10 02:14:24 +00:00
|
|
|
## Licensed under either of
|
|
|
|
## * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE))
|
|
|
|
## * MIT license ([LICENSE-MIT](LICENSE-MIT))
|
|
|
|
## at your option.
|
|
|
|
## This file may not be copied, modified, or distributed except according to
|
|
|
|
## those terms.
|
|
|
|
|
2020-05-06 16:31:47 +00:00
|
|
|
import sequtils, tables, sets, strutils
|
2020-06-07 07:15:21 +00:00
|
|
|
import chronos, chronicles, metrics
|
2019-09-10 02:15:52 +00:00
|
|
|
import pubsub,
|
|
|
|
pubsubpeer,
|
2019-12-06 02:16:18 +00:00
|
|
|
timedcache,
|
2020-07-15 19:18:55 +00:00
|
|
|
peertable,
|
2019-12-06 02:16:18 +00:00
|
|
|
rpc/[messages, message],
|
2020-06-19 17:29:43 +00:00
|
|
|
../../stream/connection,
|
2020-07-01 06:25:09 +00:00
|
|
|
../../peerid,
|
2020-07-14 00:02:16 +00:00
|
|
|
../../peerinfo
|
2019-09-10 02:14:24 +00:00
|
|
|
|
|
|
|
logScope:
|
2020-06-10 08:48:01 +00:00
|
|
|
topics = "floodsub"
|
2019-09-10 02:14:24 +00:00
|
|
|
|
|
|
|
const FloodSubCodec* = "/floodsub/1.0.0"
|
|
|
|
|
|
|
|
type
|
2019-10-03 22:22:49 +00:00
|
|
|
FloodSub* = ref object of PubSub
|
2020-07-15 19:18:55 +00:00
|
|
|
floodsub*: PeerTable # topic to remote peer map
|
|
|
|
seen*: TimedCache[string] # list of messages forwarded to peers
|
2019-12-06 02:16:18 +00:00
|
|
|
|
|
|
|
method subscribeTopic*(f: FloodSub,
|
|
|
|
topic: string,
|
|
|
|
subscribe: bool,
|
2020-05-27 18:33:49 +00:00
|
|
|
peerId: string) {.gcsafe, async.} =
|
|
|
|
await procCall PubSub(f).subscribeTopic(topic, subscribe, peerId)
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2020-07-13 13:32:38 +00:00
|
|
|
let peer = f.peers.getOrDefault(peerId)
|
|
|
|
if peer == nil:
|
|
|
|
debug "subscribeTopic on a nil peer!"
|
|
|
|
return
|
|
|
|
|
2019-12-10 20:50:35 +00:00
|
|
|
if topic notin f.floodsub:
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub[topic] = initHashSet[PubSubPeer]()
|
2019-12-10 20:50:35 +00:00
|
|
|
|
|
|
|
if subscribe:
|
2020-07-13 13:32:38 +00:00
|
|
|
trace "adding subscription for topic", peer = peer.id, name = topic
|
2019-12-10 20:50:35 +00:00
|
|
|
# subscribe the peer to the topic
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub[topic].incl(peer)
|
2019-12-10 20:50:35 +00:00
|
|
|
else:
|
2020-07-13 13:32:38 +00:00
|
|
|
trace "removing subscription for topic", peer = peer.id, name = topic
|
2019-12-10 20:50:35 +00:00
|
|
|
# unsubscribe the peer from the topic
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub[topic].excl(peer)
|
2019-12-10 20:50:35 +00:00
|
|
|
|
2020-07-08 00:33:05 +00:00
|
|
|
method handleDisconnect*(f: FloodSub, peer: PubSubPeer) =
|
2019-12-06 02:16:18 +00:00
|
|
|
## handle peer disconnects
|
2020-06-19 21:19:07 +00:00
|
|
|
for t in toSeq(f.floodsub.keys):
|
|
|
|
if t in f.floodsub:
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub[t].excl(peer)
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2020-07-08 00:33:05 +00:00
|
|
|
procCall PubSub(f).handleDisconnect(peer)
|
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
method rpcHandler*(f: FloodSub,
|
|
|
|
peer: PubSubPeer,
|
2019-12-17 05:24:03 +00:00
|
|
|
rpcMsgs: seq[RPCMsg]) {.async.} =
|
|
|
|
await procCall PubSub(f).rpcHandler(peer, rpcMsgs)
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2020-01-07 08:04:02 +00:00
|
|
|
for m in rpcMsgs: # for all RPC messages
|
2019-12-06 02:16:18 +00:00
|
|
|
if m.messages.len > 0: # if there are any messages
|
2020-07-13 13:32:38 +00:00
|
|
|
var toSendPeers = initHashSet[PubSubPeer]()
|
2019-12-06 02:16:18 +00:00
|
|
|
for msg in m.messages: # for every message
|
2020-06-28 15:56:38 +00:00
|
|
|
let msgId = f.msgIdProvider(msg)
|
|
|
|
logScope: msgId
|
|
|
|
|
|
|
|
if msgId notin f.seen:
|
|
|
|
f.seen.put(msgId) # add the message to the seen cache
|
2019-12-17 05:24:03 +00:00
|
|
|
|
2020-05-06 09:26:08 +00:00
|
|
|
if f.verifySignature and not msg.verify(peer.peerInfo):
|
2019-12-17 05:24:03 +00:00
|
|
|
trace "dropping message due to failed signature verification"
|
|
|
|
continue
|
|
|
|
|
|
|
|
if not (await f.validate(msg)):
|
|
|
|
trace "dropping message due to failed validation"
|
|
|
|
continue
|
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
for t in msg.topicIDs: # for every topic in the message
|
|
|
|
if t in f.floodsub:
|
|
|
|
toSendPeers.incl(f.floodsub[t]) # get all the peers interested in this topic
|
|
|
|
if t in f.topics: # check that we're subscribed to it
|
|
|
|
for h in f.topics[t].handler:
|
2020-06-28 15:56:38 +00:00
|
|
|
trace "calling handler for message", topicId = t,
|
2019-12-23 18:45:12 +00:00
|
|
|
localPeer = f.peerInfo.id,
|
2020-06-28 15:56:38 +00:00
|
|
|
fromPeer = msg.fromPeer.pretty
|
2020-07-08 00:33:05 +00:00
|
|
|
|
|
|
|
try:
|
|
|
|
await h(t, msg.data) # trigger user provided handler
|
|
|
|
except CatchableError as exc:
|
|
|
|
trace "exception in message handler", exc = exc.msg
|
2019-09-10 02:15:52 +00:00
|
|
|
|
2019-09-14 13:56:02 +00:00
|
|
|
# forward the message to all peers interested in it
|
2020-07-17 19:46:24 +00:00
|
|
|
let published = await f.publishHelper(toSendPeers, m.messages)
|
2020-06-17 04:14:02 +00:00
|
|
|
|
2020-07-17 19:46:24 +00:00
|
|
|
trace "forwared message to peers", peers = published
|
2019-09-10 02:15:52 +00:00
|
|
|
|
2020-06-12 23:54:12 +00:00
|
|
|
method init*(f: FloodSub) =
|
2019-12-17 05:24:03 +00:00
|
|
|
proc handler(conn: Connection, proto: string) {.async.} =
|
2019-09-12 10:08:11 +00:00
|
|
|
## main protocol handler that gets triggered on every
|
2019-09-10 02:15:52 +00:00
|
|
|
## connection for a protocol string
|
|
|
|
## e.g. ``/floodsub/1.0.0``, etc...
|
|
|
|
##
|
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
await f.handleConn(conn, proto)
|
2019-09-10 02:14:24 +00:00
|
|
|
|
|
|
|
f.handler = handler
|
2019-09-10 02:15:52 +00:00
|
|
|
f.codec = FloodSubCodec
|
|
|
|
|
2020-07-08 00:33:05 +00:00
|
|
|
method subscribePeer*(p: FloodSub,
|
|
|
|
conn: Connection) =
|
|
|
|
procCall PubSub(p).subscribePeer(conn)
|
2020-05-21 17:33:48 +00:00
|
|
|
asyncCheck p.handleConn(conn, FloodSubCodec)
|
|
|
|
|
2019-09-10 02:15:52 +00:00
|
|
|
method publish*(f: FloodSub,
|
|
|
|
topic: string,
|
2020-07-08 00:33:05 +00:00
|
|
|
data: seq[byte]): Future[int] {.async.} =
|
|
|
|
# base returns always 0
|
|
|
|
discard await procCall PubSub(f).publish(topic, data)
|
2019-10-03 22:22:49 +00:00
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
if data.len <= 0 or topic.len <= 0:
|
|
|
|
trace "topic or data missing, skipping publish"
|
|
|
|
return
|
|
|
|
|
|
|
|
if topic notin f.floodsub:
|
|
|
|
trace "missing peers for topic, skipping publish"
|
|
|
|
return
|
|
|
|
|
|
|
|
trace "publishing on topic", name = topic
|
2020-07-15 03:51:33 +00:00
|
|
|
inc f.msgSeqno
|
|
|
|
let msg = Message.init(f.peerInfo, data, topic, f.msgSeqno, f.sign)
|
2020-07-17 19:46:24 +00:00
|
|
|
|
2020-04-11 04:08:25 +00:00
|
|
|
# start the future but do not wait yet
|
2020-07-17 19:46:24 +00:00
|
|
|
let published = await f.publishHelper(f.floodsub.getOrDefault(topic), @[msg])
|
2019-09-10 02:14:24 +00:00
|
|
|
|
2020-06-17 04:14:02 +00:00
|
|
|
libp2p_pubsub_messages_published.inc(labelValues = [topic])
|
|
|
|
|
2020-07-17 19:46:24 +00:00
|
|
|
trace "published message to peers", peers = published,
|
2020-07-08 00:33:05 +00:00
|
|
|
msg = msg.shortLog()
|
2020-07-17 19:46:24 +00:00
|
|
|
return published
|
2020-07-08 00:33:05 +00:00
|
|
|
|
2019-12-10 20:50:35 +00:00
|
|
|
method unsubscribe*(f: FloodSub,
|
2019-12-17 05:24:03 +00:00
|
|
|
topics: seq[TopicPair]) {.async.} =
|
2019-09-10 02:15:52 +00:00
|
|
|
await procCall PubSub(f).unsubscribe(topics)
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2019-09-10 02:15:52 +00:00
|
|
|
for p in f.peers.values:
|
2019-09-24 16:16:39 +00:00
|
|
|
await f.sendSubs(p, topics.mapIt(it.topic).deduplicate(), false)
|
2019-09-10 02:14:24 +00:00
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
method initPubSub*(f: FloodSub) =
|
2020-04-30 13:22:31 +00:00
|
|
|
procCall PubSub(f).initPubSub()
|
2019-10-03 22:22:49 +00:00
|
|
|
f.peers = initTable[string, PubSubPeer]()
|
|
|
|
f.topics = initTable[string, Topic]()
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub = initTable[string, HashSet[PubSubPeer]]()
|
2019-12-06 02:16:18 +00:00
|
|
|
f.seen = newTimedCache[string](2.minutes)
|
2019-10-03 22:22:49 +00:00
|
|
|
f.init()
|