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-09-04 06:10:32 +00:00
|
|
|
import std/[sequtils, sets, tables]
|
2020-06-07 07:15:21 +00:00
|
|
|
import chronos, chronicles, metrics
|
2020-09-04 06:10:32 +00:00
|
|
|
import ./pubsub,
|
|
|
|
./pubsubpeer,
|
|
|
|
./timedcache,
|
|
|
|
./peertable,
|
|
|
|
./rpc/[message, messages],
|
2020-06-19 17:29:43 +00:00
|
|
|
../../stream/connection,
|
2020-07-01 06:25:09 +00:00
|
|
|
../../peerid,
|
2020-09-04 06:10:32 +00:00
|
|
|
../../peerinfo,
|
|
|
|
../../utility
|
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-08-17 10:10:22 +00:00
|
|
|
peer: PubsubPeer) {.gcsafe.} =
|
|
|
|
procCall PubSub(f).subscribeTopic(topic, subscribe, peer)
|
|
|
|
|
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-09-06 08:31:47 +00:00
|
|
|
trace "adding subscription for topic", peer, 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-09-06 08:31:47 +00:00
|
|
|
trace "removing subscription for topic", peer, 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-08-12 00:05:49 +00:00
|
|
|
method unsubscribePeer*(f: FloodSub, peer: PeerID) =
|
2019-12-06 02:16:18 +00:00
|
|
|
## handle peer disconnects
|
2020-08-06 02:12:52 +00:00
|
|
|
##
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "unsubscribing floodsub peer", peer
|
2020-08-12 00:05:49 +00:00
|
|
|
let pubSubPeer = f.peers.getOrDefault(peer)
|
|
|
|
if pubSubPeer.isNil:
|
|
|
|
return
|
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
for _, v in f.floodsub.mpairs():
|
|
|
|
v.excl(pubSubPeer)
|
2020-08-12 00:05:49 +00:00
|
|
|
|
|
|
|
procCall PubSub(f).unsubscribePeer(peer)
|
2020-08-06 02:12:52 +00:00
|
|
|
|
2019-12-06 02:16:18 +00:00
|
|
|
method rpcHandler*(f: FloodSub,
|
|
|
|
peer: PubSubPeer,
|
2020-09-01 07:33:03 +00:00
|
|
|
rpcMsg: RPCMsg) {.async.} =
|
|
|
|
await procCall PubSub(f).rpcHandler(peer, rpcMsg)
|
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
for msg in rpcMsg.messages: # for every message
|
|
|
|
let msgId = f.msgIdProvider(msg)
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
if f.seen.put(msgId):
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Dropping already-seen message", msgId, peer
|
2020-09-04 06:10:32 +00:00
|
|
|
continue
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-23 15:56:33 +00:00
|
|
|
if (msg.signature.len > 0 or f.verifySignature) and not msg.verify():
|
|
|
|
# always validate if signature is present or required
|
2020-09-06 08:31:47 +00:00
|
|
|
debug "Dropping message due to failed signature verification", msgId, peer
|
2020-09-04 06:10:32 +00:00
|
|
|
continue
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-23 15:56:33 +00:00
|
|
|
if msg.seqno.len > 0 and msg.seqno.len != 8:
|
|
|
|
# if we have seqno should be 8 bytes long
|
|
|
|
debug "Dropping message due to invalid seqno length", msgId, peer
|
|
|
|
continue
|
|
|
|
|
|
|
|
# g.anonymize needs no evaluation when receiving messages
|
|
|
|
# as we have a "lax" policy and allow signed messages
|
|
|
|
|
2020-10-21 03:25:42 +00:00
|
|
|
let validation = await f.validate(msg)
|
|
|
|
case validation
|
|
|
|
of ValidationResult.Reject, ValidationResult.Ignore:
|
|
|
|
debug "Dropping message due to ignored validation", msgId, peer
|
2020-09-04 06:10:32 +00:00
|
|
|
continue
|
2020-10-21 03:25:42 +00:00
|
|
|
of ValidationResult.Accept:
|
|
|
|
discard
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
var toSendPeers = initHashSet[PubSubPeer]()
|
|
|
|
for t in msg.topicIDs: # for every topic in the message
|
|
|
|
f.floodsub.withValue(t, peers): toSendPeers.incl(peers[])
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
await handleData(f, t, msg.data)
|
2020-09-01 07:33:03 +00:00
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
# In theory, if topics are the same in all messages, we could batch - we'd
|
|
|
|
# also have to be careful to only include validated messages
|
|
|
|
f.broadcast(toSeq(toSendPeers), RPCMsg(messages: @[msg]))
|
|
|
|
trace "Forwared message to peers", peers = toSendPeers.len
|
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...
|
|
|
|
##
|
2020-09-04 16:30:45 +00:00
|
|
|
try:
|
|
|
|
await f.handleConn(conn, proto)
|
|
|
|
except CancelledError:
|
|
|
|
# This is top-level procedure which will work as separate task, so it
|
|
|
|
# do not need to propogate CancelledError.
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Unexpected cancellation in floodsub handler", conn
|
2020-09-04 16:30:45 +00:00
|
|
|
except CatchableError as exc:
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "FloodSub handler leaks an error", exc = exc.msg, conn
|
2019-09-10 02:14:24 +00:00
|
|
|
|
|
|
|
f.handler = handler
|
2019-09-10 02:15:52 +00:00
|
|
|
f.codec = FloodSubCodec
|
|
|
|
|
|
|
|
method publish*(f: FloodSub,
|
|
|
|
topic: string,
|
2020-09-01 07:33:03 +00:00
|
|
|
data: seq[byte]): Future[int] {.async.} =
|
2020-07-08 00:33:05 +00:00
|
|
|
# base returns always 0
|
2020-09-01 07:33:03 +00:00
|
|
|
discard await procCall PubSub(f).publish(topic, data)
|
2019-10-03 22:22:49 +00:00
|
|
|
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Publishing message on topic", data = data.shortLog, topic
|
2020-09-04 06:10:32 +00:00
|
|
|
|
|
|
|
if topic.len <= 0: # data could be 0/empty
|
2020-09-06 08:31:47 +00:00
|
|
|
debug "Empty topic, skipping publish", topic
|
2020-09-01 07:33:03 +00:00
|
|
|
return 0
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2020-09-04 06:10:32 +00:00
|
|
|
let peers = toSeq(f.floodsub.getOrDefault(topic))
|
|
|
|
|
|
|
|
if peers.len == 0:
|
2020-09-06 08:31:47 +00:00
|
|
|
debug "No peers for topic, skipping publish", topic
|
2020-09-04 06:10:32 +00:00
|
|
|
return 0
|
2019-12-06 02:16:18 +00:00
|
|
|
|
2020-07-15 03:51:33 +00:00
|
|
|
inc f.msgSeqno
|
2020-09-01 07:33:03 +00:00
|
|
|
let
|
2020-09-23 15:56:33 +00:00
|
|
|
msg =
|
|
|
|
if f.anonymize:
|
|
|
|
Message.init(none(PeerInfo), data, topic, none(uint64), false)
|
|
|
|
else:
|
|
|
|
Message.init(some(f.peerInfo), data, topic, some(f.msgSeqno), f.sign)
|
2020-09-04 06:10:32 +00:00
|
|
|
msgId = f.msgIdProvider(msg)
|
2020-07-17 19:46:24 +00:00
|
|
|
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Created new message",
|
|
|
|
msg = shortLog(msg), peers = peers.len, topic, msgId
|
2020-09-04 06:10:32 +00:00
|
|
|
|
|
|
|
if f.seen.put(msgId):
|
|
|
|
# custom msgid providers might cause this
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Dropping already-seen message", msgId, topic
|
2020-09-04 06:10:32 +00:00
|
|
|
return 0
|
|
|
|
|
|
|
|
# Try to send to all peers that are known to be interested
|
|
|
|
f.broadcast(peers, RPCMsg(messages: @[msg]))
|
2019-09-10 02:14:24 +00:00
|
|
|
|
2020-08-04 23:27:59 +00:00
|
|
|
when defined(libp2p_expensive_metrics):
|
|
|
|
libp2p_pubsub_messages_published.inc(labelValues = [topic])
|
2020-06-17 04:14:02 +00:00
|
|
|
|
2020-09-06 08:31:47 +00:00
|
|
|
trace "Published message to peers", msgId, topic
|
2020-09-04 06:10:32 +00:00
|
|
|
|
2020-09-01 07:33:03 +00:00
|
|
|
return peers.len
|
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:
|
2020-09-01 07:33:03 +00:00
|
|
|
f.sendSubs(p, topics.mapIt(it.topic).deduplicate(), false)
|
2019-09-10 02:14:24 +00:00
|
|
|
|
2020-07-20 16:16:13 +00:00
|
|
|
method unsubscribeAll*(f: FloodSub, topic: string) {.async.} =
|
|
|
|
await procCall PubSub(f).unsubscribeAll(topic)
|
|
|
|
|
|
|
|
for p in f.peers.values:
|
2020-09-01 07:33:03 +00:00
|
|
|
f.sendSubs(p, @[topic], false)
|
2020-07-20 16:16:13 +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()
|
2020-07-13 13:32:38 +00:00
|
|
|
f.floodsub = initTable[string, HashSet[PubSubPeer]]()
|
2020-09-04 06:10:32 +00:00
|
|
|
f.seen = TimedCache[string].init(2.minutes)
|
2019-10-03 22:22:49 +00:00
|
|
|
f.init()
|