2021-04-28 20:10:44 +00:00
|
|
|
package relay
|
|
|
|
|
|
|
|
import (
|
|
|
|
"context"
|
2021-07-29 22:08:53 +00:00
|
|
|
"crypto/sha256"
|
2021-04-28 20:10:44 +00:00
|
|
|
"errors"
|
2021-11-01 14:42:55 +00:00
|
|
|
"fmt"
|
2021-04-28 20:10:44 +00:00
|
|
|
"sync"
|
|
|
|
|
|
|
|
proto "github.com/golang/protobuf/proto"
|
|
|
|
"github.com/libp2p/go-libp2p-core/host"
|
2021-07-29 22:08:53 +00:00
|
|
|
"github.com/libp2p/go-libp2p-core/protocol"
|
2021-11-01 14:42:55 +00:00
|
|
|
"go.opencensus.io/stats"
|
|
|
|
"go.opencensus.io/tag"
|
2022-01-18 18:17:06 +00:00
|
|
|
"go.uber.org/zap"
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2021-11-01 14:42:55 +00:00
|
|
|
pubsub "github.com/libp2p/go-libp2p-pubsub"
|
2021-07-29 22:08:53 +00:00
|
|
|
pubsub_pb "github.com/libp2p/go-libp2p-pubsub/pb"
|
2021-11-01 14:42:55 +00:00
|
|
|
v2 "github.com/status-im/go-waku/waku/v2"
|
|
|
|
"github.com/status-im/go-waku/waku/v2/metrics"
|
|
|
|
waku_proto "github.com/status-im/go-waku/waku/v2/protocol"
|
2021-04-28 20:10:44 +00:00
|
|
|
"github.com/status-im/go-waku/waku/v2/protocol/pb"
|
|
|
|
)
|
|
|
|
|
2021-07-29 22:08:53 +00:00
|
|
|
const WakuRelayID_v200 = protocol.ID("/vac/waku/relay/2.0.0")
|
2021-11-10 14:28:45 +00:00
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
var DefaultWakuTopic string = waku_proto.DefaultPubsubTopic().String()
|
2021-04-28 20:10:44 +00:00
|
|
|
|
|
|
|
type WakuRelay struct {
|
|
|
|
host host.Host
|
2021-07-29 22:08:53 +00:00
|
|
|
pubsub *pubsub.PubSub
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2022-01-18 18:17:06 +00:00
|
|
|
log *zap.SugaredLogger
|
|
|
|
|
2021-11-06 22:46:58 +00:00
|
|
|
bcaster v2.Broadcaster
|
|
|
|
|
2021-12-06 08:43:00 +00:00
|
|
|
minPeersToPublish int
|
|
|
|
|
2021-11-06 22:46:58 +00:00
|
|
|
// TODO: convert to concurrent maps
|
2021-04-28 20:10:44 +00:00
|
|
|
topicsMutex sync.Mutex
|
2021-11-19 16:19:48 +00:00
|
|
|
wakuRelayTopics map[string]*pubsub.Topic
|
|
|
|
relaySubs map[string]*pubsub.Subscription
|
2021-11-01 14:42:55 +00:00
|
|
|
|
2021-11-06 22:46:58 +00:00
|
|
|
// TODO: convert to concurrent maps
|
2021-11-19 16:19:48 +00:00
|
|
|
subscriptions map[string][]*Subscription
|
2021-11-01 14:42:55 +00:00
|
|
|
subscriptionsMutex sync.Mutex
|
2021-07-29 22:08:53 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
func msgIdFn(pmsg *pubsub_pb.Message) string {
|
|
|
|
hash := sha256.Sum256(pmsg.Data)
|
|
|
|
return string(hash[:])
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// NewWakuRelay returns a new instance of a WakuRelay struct
|
2022-01-18 18:17:06 +00:00
|
|
|
func NewWakuRelay(ctx context.Context, h host.Host, bcaster v2.Broadcaster, minPeersToPublish int, log *zap.SugaredLogger, opts ...pubsub.Option) (*WakuRelay, error) {
|
2021-04-28 20:10:44 +00:00
|
|
|
w := new(WakuRelay)
|
|
|
|
w.host = h
|
2021-11-19 16:19:48 +00:00
|
|
|
w.wakuRelayTopics = make(map[string]*pubsub.Topic)
|
|
|
|
w.relaySubs = make(map[string]*pubsub.Subscription)
|
|
|
|
w.subscriptions = make(map[string][]*Subscription)
|
2021-11-01 14:42:55 +00:00
|
|
|
w.bcaster = bcaster
|
2021-12-06 08:43:00 +00:00
|
|
|
w.minPeersToPublish = minPeersToPublish
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log = log.Named("relay")
|
2021-07-29 22:08:53 +00:00
|
|
|
|
|
|
|
// default options required by WakuRelay
|
|
|
|
opts = append(opts, pubsub.WithMessageSignaturePolicy(pubsub.StrictNoSign))
|
|
|
|
opts = append(opts, pubsub.WithNoAuthor())
|
|
|
|
opts = append(opts, pubsub.WithMessageIdFn(msgIdFn))
|
|
|
|
|
|
|
|
opts = append(opts, pubsub.WithGossipSubProtocols(
|
|
|
|
[]protocol.ID{pubsub.GossipSubID_v11, pubsub.GossipSubID_v10, pubsub.FloodSubID, WakuRelayID_v200},
|
2021-10-08 13:50:56 +00:00
|
|
|
func(feat pubsub.GossipSubFeature, proto protocol.ID) bool {
|
2021-07-29 22:08:53 +00:00
|
|
|
switch feat {
|
2021-10-08 13:50:56 +00:00
|
|
|
case pubsub.GossipSubFeatureMesh:
|
|
|
|
return proto == pubsub.GossipSubID_v11 || proto == pubsub.GossipSubID_v10
|
|
|
|
case pubsub.GossipSubFeaturePX:
|
|
|
|
return proto == pubsub.GossipSubID_v11
|
2021-07-29 22:08:53 +00:00
|
|
|
default:
|
|
|
|
return false
|
|
|
|
}
|
2021-10-08 13:50:56 +00:00
|
|
|
},
|
2021-07-29 22:08:53 +00:00
|
|
|
))
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2021-07-29 22:08:53 +00:00
|
|
|
ps, err := pubsub.NewGossipSub(ctx, h, opts...)
|
2021-04-28 20:10:44 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
w.pubsub = ps
|
|
|
|
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Info("Relay protocol started")
|
2021-04-28 20:10:44 +00:00
|
|
|
|
|
|
|
return w, nil
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// PubSub returns the implementation of the pubsub system
|
2021-07-29 22:08:53 +00:00
|
|
|
func (w *WakuRelay) PubSub() *pubsub.PubSub {
|
2021-04-28 20:10:44 +00:00
|
|
|
return w.pubsub
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// Topics returns a list of all the pubsub topics currently subscribed to
|
2021-11-19 16:19:48 +00:00
|
|
|
func (w *WakuRelay) Topics() []string {
|
2021-04-28 20:10:44 +00:00
|
|
|
defer w.topicsMutex.Unlock()
|
|
|
|
w.topicsMutex.Lock()
|
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
var result []string
|
|
|
|
for topic := range w.relaySubs {
|
2021-04-28 20:10:44 +00:00
|
|
|
result = append(result, topic)
|
|
|
|
}
|
|
|
|
return result
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// SetPubSub is used to set an aimplementation of the pubsub system
|
2021-07-29 22:08:53 +00:00
|
|
|
func (w *WakuRelay) SetPubSub(pubSub *pubsub.PubSub) {
|
2021-04-28 20:10:44 +00:00
|
|
|
w.pubsub = pubSub
|
|
|
|
}
|
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
func (w *WakuRelay) upsertTopic(topic string) (*pubsub.Topic, error) {
|
2021-04-28 20:10:44 +00:00
|
|
|
defer w.topicsMutex.Unlock()
|
|
|
|
w.topicsMutex.Lock()
|
|
|
|
|
|
|
|
pubSubTopic, ok := w.wakuRelayTopics[topic]
|
|
|
|
if !ok { // Joins topic if node hasn't joined yet
|
|
|
|
newTopic, err := w.pubsub.Join(string(topic))
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
w.wakuRelayTopics[topic] = newTopic
|
|
|
|
pubSubTopic = newTopic
|
|
|
|
}
|
|
|
|
return pubSubTopic, nil
|
|
|
|
}
|
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
func (w *WakuRelay) subscribe(topic string) (subs *pubsub.Subscription, err error) {
|
2021-04-28 20:10:44 +00:00
|
|
|
sub, ok := w.relaySubs[topic]
|
|
|
|
if !ok {
|
|
|
|
pubSubTopic, err := w.upsertTopic(topic)
|
|
|
|
if err != nil {
|
2021-11-01 14:42:55 +00:00
|
|
|
return nil, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
sub, err = pubSubTopic.Subscribe()
|
|
|
|
if err != nil {
|
2021-11-01 14:42:55 +00:00
|
|
|
return nil, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
w.relaySubs[topic] = sub
|
|
|
|
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Info("Subscribing to topic ", topic)
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2021-11-01 14:42:55 +00:00
|
|
|
return sub, nil
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// PublishToTopic is used to broadcast a WakuMessage to a pubsub topic
|
2021-11-20 00:03:05 +00:00
|
|
|
func (w *WakuRelay) PublishToTopic(ctx context.Context, message *pb.WakuMessage, topic string) ([]byte, error) {
|
2021-04-28 20:10:44 +00:00
|
|
|
// Publish a `WakuMessage` to a PubSub topic.
|
|
|
|
if w.pubsub == nil {
|
2021-07-29 12:40:54 +00:00
|
|
|
return nil, errors.New("PubSub hasn't been set")
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if message == nil {
|
|
|
|
return nil, errors.New("message can't be null")
|
|
|
|
}
|
|
|
|
|
2021-12-06 08:43:00 +00:00
|
|
|
if !w.EnoughPeersToPublishToTopic(topic) {
|
|
|
|
return nil, errors.New("not enougth peers to publish")
|
|
|
|
}
|
|
|
|
|
2021-11-19 20:01:52 +00:00
|
|
|
pubSubTopic, err := w.upsertTopic(topic)
|
2021-04-28 20:10:44 +00:00
|
|
|
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
|
|
|
out, err := proto.Marshal(message)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
|
|
|
err = pubSubTopic.Publish(ctx, out)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
|
|
|
hash := pb.Hash(out)
|
|
|
|
|
|
|
|
return hash, nil
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// Publish is used to broadcast a WakuMessage to the default waku pubsub topic
|
2021-11-19 20:01:52 +00:00
|
|
|
func (w *WakuRelay) Publish(ctx context.Context, message *pb.WakuMessage) ([]byte, error) {
|
2021-11-20 00:03:05 +00:00
|
|
|
return w.PublishToTopic(ctx, message, DefaultWakuTopic)
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
2021-10-11 22:45:54 +00:00
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// Stop unmounts the relay protocol and stops all subscriptions
|
2021-10-11 22:45:54 +00:00
|
|
|
func (w *WakuRelay) Stop() {
|
|
|
|
w.host.RemoveStreamHandler(WakuRelayID_v200)
|
2021-11-01 14:42:55 +00:00
|
|
|
w.subscriptionsMutex.Lock()
|
|
|
|
defer w.subscriptionsMutex.Unlock()
|
|
|
|
|
|
|
|
for _, topic := range w.Topics() {
|
|
|
|
for _, sub := range w.subscriptions[topic] {
|
|
|
|
sub.Unsubscribe()
|
|
|
|
}
|
|
|
|
}
|
|
|
|
w.subscriptions = nil
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// EnoughPeersToPublish returns whether there are enough peers connected in the default waku pubsub topic
|
2021-12-06 08:43:00 +00:00
|
|
|
func (w *WakuRelay) EnoughPeersToPublish() bool {
|
|
|
|
return w.EnoughPeersToPublishToTopic(DefaultWakuTopic)
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// EnoughPeersToPublish returns whether there are enough peers connected in a pubsub topic
|
2021-12-06 08:43:00 +00:00
|
|
|
func (w *WakuRelay) EnoughPeersToPublishToTopic(topic string) bool {
|
|
|
|
return len(w.PubSub().ListPeers(topic)) >= w.minPeersToPublish
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// SubscribeToTopic returns a Subscription to receive messages from a pubsub topic
|
2021-11-20 00:03:05 +00:00
|
|
|
func (w *WakuRelay) SubscribeToTopic(ctx context.Context, topic string) (*Subscription, error) {
|
2021-11-01 14:42:55 +00:00
|
|
|
// Subscribes to a PubSub topic.
|
|
|
|
// NOTE The data field SHOULD be decoded as a WakuMessage.
|
2021-11-19 20:01:52 +00:00
|
|
|
sub, err := w.subscribe(topic)
|
2021-11-01 14:42:55 +00:00
|
|
|
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
|
|
|
// Create client subscription
|
|
|
|
subscription := new(Subscription)
|
|
|
|
subscription.closed = false
|
|
|
|
subscription.C = make(chan *waku_proto.Envelope, 1024) // To avoid blocking
|
|
|
|
subscription.quit = make(chan struct{})
|
|
|
|
|
|
|
|
w.subscriptionsMutex.Lock()
|
|
|
|
defer w.subscriptionsMutex.Unlock()
|
|
|
|
|
2021-11-19 20:01:52 +00:00
|
|
|
w.subscriptions[topic] = append(w.subscriptions[topic], subscription)
|
2021-11-01 14:42:55 +00:00
|
|
|
|
|
|
|
if w.bcaster != nil {
|
2022-04-25 19:31:26 +00:00
|
|
|
w.bcaster.Register(&topic, subscription.C)
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
|
|
|
|
2021-11-19 20:01:52 +00:00
|
|
|
go w.subscribeToTopic(topic, subscription, sub)
|
2021-11-01 14:42:55 +00:00
|
|
|
|
|
|
|
return subscription, nil
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// SubscribeToTopic returns a Subscription to receive messages from the default waku pubsub topic
|
2021-11-19 20:01:52 +00:00
|
|
|
func (w *WakuRelay) Subscribe(ctx context.Context) (*Subscription, error) {
|
2021-11-20 00:03:05 +00:00
|
|
|
return w.SubscribeToTopic(ctx, DefaultWakuTopic)
|
2021-11-19 20:01:52 +00:00
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// Unsubscribe closes a subscription to a pubsub topic
|
2021-11-19 16:19:48 +00:00
|
|
|
func (w *WakuRelay) Unsubscribe(ctx context.Context, topic string) error {
|
|
|
|
if _, ok := w.relaySubs[topic]; !ok {
|
2021-11-06 10:49:47 +00:00
|
|
|
return fmt.Errorf("topics %s is not subscribed", (string)(topic))
|
|
|
|
}
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Info("Unsubscribing from topic ", topic)
|
2021-11-06 10:49:47 +00:00
|
|
|
|
|
|
|
for _, sub := range w.subscriptions[topic] {
|
|
|
|
sub.Unsubscribe()
|
|
|
|
}
|
|
|
|
|
|
|
|
w.relaySubs[topic].Cancel()
|
|
|
|
delete(w.relaySubs, topic)
|
|
|
|
|
|
|
|
err := w.wakuRelayTopics[topic].Close()
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
delete(w.wakuRelayTopics, topic)
|
|
|
|
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2021-11-01 14:42:55 +00:00
|
|
|
func (w *WakuRelay) nextMessage(ctx context.Context, sub *pubsub.Subscription) <-chan *pubsub.Message {
|
|
|
|
msgChannel := make(chan *pubsub.Message, 1024)
|
|
|
|
go func(msgChannel chan *pubsub.Message) {
|
2021-11-06 10:49:47 +00:00
|
|
|
defer func() {
|
|
|
|
if r := recover(); r != nil {
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Debug("recovered msgChannel")
|
2021-11-06 10:49:47 +00:00
|
|
|
}
|
|
|
|
}()
|
|
|
|
|
2021-11-01 14:42:55 +00:00
|
|
|
for {
|
|
|
|
msg, err := sub.Next(ctx)
|
|
|
|
if err != nil {
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Error(fmt.Errorf("subscription failed: %w", err))
|
2021-11-01 14:42:55 +00:00
|
|
|
sub.Cancel()
|
|
|
|
close(msgChannel)
|
2021-11-19 16:19:48 +00:00
|
|
|
for _, subscription := range w.subscriptions[sub.Topic()] {
|
2021-11-01 14:42:55 +00:00
|
|
|
subscription.Unsubscribe()
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
msgChannel <- msg
|
|
|
|
}
|
|
|
|
}(msgChannel)
|
|
|
|
return msgChannel
|
|
|
|
}
|
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
func (w *WakuRelay) subscribeToTopic(t string, subscription *Subscription, sub *pubsub.Subscription) {
|
2021-11-01 14:42:55 +00:00
|
|
|
ctx, err := tag.New(context.Background(), tag.Insert(metrics.KeyType, "relay"))
|
|
|
|
if err != nil {
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Error(err)
|
2021-11-01 14:42:55 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
subChannel := w.nextMessage(ctx, sub)
|
|
|
|
|
|
|
|
for {
|
|
|
|
select {
|
|
|
|
case <-subscription.quit:
|
2022-04-25 19:31:26 +00:00
|
|
|
func(topic string) {
|
2022-02-23 15:08:27 +00:00
|
|
|
subscription.Lock()
|
|
|
|
defer subscription.Unlock()
|
|
|
|
|
|
|
|
if subscription.closed {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
subscription.closed = true
|
|
|
|
if w.bcaster != nil {
|
2022-04-25 19:31:26 +00:00
|
|
|
<-w.bcaster.WaitUnregister(&topic, subscription.C) // Remove from broadcast list
|
2022-02-23 15:08:27 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
close(subscription.C)
|
2022-04-25 19:31:26 +00:00
|
|
|
}(t)
|
2021-11-01 14:42:55 +00:00
|
|
|
// TODO: if there are no more relay subscriptions, close the pubsub subscription
|
|
|
|
case msg := <-subChannel:
|
2021-11-06 10:49:47 +00:00
|
|
|
if msg == nil {
|
|
|
|
return
|
|
|
|
}
|
2021-11-01 14:42:55 +00:00
|
|
|
stats.Record(ctx, metrics.Messages.M(1))
|
|
|
|
wakuMessage := &pb.WakuMessage{}
|
|
|
|
if err := proto.Unmarshal(msg.Data, wakuMessage); err != nil {
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Error("could not decode message", err)
|
2021-11-01 14:42:55 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
envelope := waku_proto.NewEnvelope(wakuMessage, string(t))
|
|
|
|
|
|
|
|
if w.bcaster != nil {
|
|
|
|
w.bcaster.Submit(envelope)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2021-10-11 22:45:54 +00:00
|
|
|
}
|