2021-04-28 20:10:44 +00:00
|
|
|
package relay
|
|
|
|
|
|
|
|
import (
|
|
|
|
"context"
|
|
|
|
"errors"
|
|
|
|
"sync"
|
|
|
|
|
2023-07-05 19:17:43 +00:00
|
|
|
"github.com/libp2p/go-libp2p/core/event"
|
2022-10-19 19:39:32 +00:00
|
|
|
"github.com/libp2p/go-libp2p/core/host"
|
2023-06-06 18:09:44 +00:00
|
|
|
"github.com/libp2p/go-libp2p/core/peer"
|
2022-10-19 19:39:32 +00:00
|
|
|
"github.com/libp2p/go-libp2p/core/protocol"
|
2023-07-05 19:17:43 +00:00
|
|
|
"github.com/libp2p/go-libp2p/p2p/host/eventbus"
|
2023-08-16 01:40:00 +00:00
|
|
|
"github.com/prometheus/client_golang/prometheus"
|
2022-01-18 18:17:06 +00:00
|
|
|
"go.uber.org/zap"
|
2023-02-06 22:16:20 +00:00
|
|
|
proto "google.golang.org/protobuf/proto"
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2021-11-01 14:42:55 +00:00
|
|
|
pubsub "github.com/libp2p/go-libp2p-pubsub"
|
2022-11-03 13:53:33 +00:00
|
|
|
"github.com/waku-org/go-waku/logging"
|
2024-02-08 09:54:58 +00:00
|
|
|
wps "github.com/waku-org/go-waku/waku/v2/peerstore"
|
2022-11-09 19:53:01 +00:00
|
|
|
waku_proto "github.com/waku-org/go-waku/waku/v2/protocol"
|
|
|
|
"github.com/waku-org/go-waku/waku/v2/protocol/pb"
|
2023-11-07 17:13:19 +00:00
|
|
|
"github.com/waku-org/go-waku/waku/v2/service"
|
2022-12-09 03:08:04 +00:00
|
|
|
"github.com/waku-org/go-waku/waku/v2/timesource"
|
2023-11-09 11:40:19 +00:00
|
|
|
"github.com/waku-org/go-waku/waku/v2/utils"
|
2021-04-28 20:10:44 +00:00
|
|
|
)
|
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// WakuRelayID_v200 is the current protocol ID used for WakuRelay
|
2021-07-29 22:08:53 +00:00
|
|
|
const WakuRelayID_v200 = protocol.ID("/vac/waku/relay/2.0.0")
|
2023-11-07 17:13:19 +00:00
|
|
|
const WakuRelayENRField = uint8(1 << 0)
|
2021-11-10 14:28:45 +00:00
|
|
|
|
2024-01-03 01:36:41 +00:00
|
|
|
const defaultMaxMsgSizeBytes = 150 * 1024
|
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// DefaultWakuTopic is the default pubsub topic used across all Waku protocols
|
2023-10-30 14:56:26 +00:00
|
|
|
var DefaultWakuTopic string = waku_proto.DefaultPubsubTopic{}.String()
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// WakuRelay is the implementation of the Waku Relay protocol
|
2021-04-28 20:10:44 +00:00
|
|
|
type WakuRelay struct {
|
2023-06-07 20:18:37 +00:00
|
|
|
host host.Host
|
2024-01-03 01:36:41 +00:00
|
|
|
relayParams *relayParameters
|
2023-06-07 20:18:37 +00:00
|
|
|
pubsub *pubsub.PubSub
|
|
|
|
params pubsub.GossipSubParams
|
|
|
|
peerScoreParams *pubsub.PeerScoreParams
|
|
|
|
peerScoreThresholds *pubsub.PeerScoreThresholds
|
|
|
|
topicParams *pubsub.TopicScoreParams
|
|
|
|
timesource timesource.Timesource
|
2023-08-16 01:40:00 +00:00
|
|
|
metrics Metrics
|
2024-01-03 01:36:41 +00:00
|
|
|
log *zap.Logger
|
|
|
|
logMessages *zap.Logger
|
2022-01-18 18:17:06 +00:00
|
|
|
|
2023-05-05 09:49:15 +00:00
|
|
|
bcaster Broadcaster
|
2021-11-06 22:46:58 +00:00
|
|
|
|
2021-12-06 08:43:00 +00:00
|
|
|
minPeersToPublish int
|
|
|
|
|
2023-09-07 21:39:10 +00:00
|
|
|
topicValidatorMutex sync.RWMutex
|
|
|
|
topicValidators map[string][]validatorFn
|
|
|
|
defaultTopicValidators []validatorFn
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
topicsMutex sync.RWMutex
|
|
|
|
topics map[string]*pubsubTopicSubscriptionDetails
|
2021-11-01 14:42:55 +00:00
|
|
|
|
2023-07-05 19:17:43 +00:00
|
|
|
events event.Bus
|
|
|
|
emitters struct {
|
|
|
|
EvtRelaySubscribed event.Emitter
|
|
|
|
EvtRelayUnsubscribed event.Emitter
|
2023-09-19 06:05:29 +00:00
|
|
|
EvtPeerTopic event.Emitter
|
2023-07-05 19:17:43 +00:00
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
|
2023-11-07 17:13:19 +00:00
|
|
|
*service.CommonService
|
2021-07-29 22:08:53 +00:00
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
type pubsubTopicSubscriptionDetails struct {
|
|
|
|
topic *pubsub.Topic
|
|
|
|
subscription *pubsub.Subscription
|
|
|
|
topicEventHandler *pubsub.TopicEventHandler
|
|
|
|
contentSubs map[int]*Subscription
|
|
|
|
}
|
|
|
|
|
2022-05-04 21:08:24 +00:00
|
|
|
// NewWakuRelay returns a new instance of a WakuRelay struct
|
2023-10-20 19:56:18 +00:00
|
|
|
func NewWakuRelay(bcaster Broadcaster, minPeersToPublish int, timesource timesource.Timesource,
|
2024-01-03 01:36:41 +00:00
|
|
|
reg prometheus.Registerer, log *zap.Logger, opts ...RelayOption) *WakuRelay {
|
2021-04-28 20:10:44 +00:00
|
|
|
w := new(WakuRelay)
|
2022-12-09 03:08:04 +00:00
|
|
|
w.timesource = timesource
|
2023-11-21 17:27:50 +00:00
|
|
|
w.topics = make(map[string]*pubsubTopicSubscriptionDetails)
|
2023-09-07 21:39:10 +00:00
|
|
|
w.topicValidators = make(map[string][]validatorFn)
|
2021-11-01 14:42:55 +00:00
|
|
|
w.bcaster = bcaster
|
2021-12-06 08:43:00 +00:00
|
|
|
w.minPeersToPublish = minPeersToPublish
|
2023-11-07 17:13:19 +00:00
|
|
|
w.CommonService = service.NewCommonService()
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log = log.Named("relay")
|
2023-11-09 11:40:19 +00:00
|
|
|
w.logMessages = utils.MessagesLogger("relay")
|
2023-07-05 19:17:43 +00:00
|
|
|
w.events = eventbus.NewBus()
|
2023-11-09 11:40:19 +00:00
|
|
|
w.metrics = newMetrics(reg, w.logMessages)
|
2024-01-03 01:36:41 +00:00
|
|
|
w.relayParams = new(relayParameters)
|
|
|
|
w.relayParams.pubsubOpts = w.defaultPubsubOptions()
|
2021-07-29 22:08:53 +00:00
|
|
|
|
2024-01-03 01:36:41 +00:00
|
|
|
options := defaultOptions()
|
|
|
|
options = append(options, opts...)
|
|
|
|
for _, opt := range options {
|
|
|
|
opt(w.relayParams)
|
|
|
|
}
|
|
|
|
w.log.Info("relay config", zap.Int("max-msg-size-bytes", w.relayParams.maxMsgSizeBytes),
|
|
|
|
zap.Int("min-peers-to-publish", w.minPeersToPublish))
|
2023-01-06 22:37:57 +00:00
|
|
|
return w
|
|
|
|
}
|
|
|
|
|
2023-06-07 20:18:37 +00:00
|
|
|
func (w *WakuRelay) peerScoreInspector(peerScoresSnapshots map[peer.ID]*pubsub.PeerScoreSnapshot) {
|
|
|
|
if w.host == nil {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
for pid, snap := range peerScoresSnapshots {
|
|
|
|
if snap.Score < w.peerScoreThresholds.GraylistThreshold {
|
|
|
|
// Disconnect bad peers
|
|
|
|
err := w.host.Network().ClosePeer(pid)
|
|
|
|
if err != nil {
|
|
|
|
w.log.Error("could not disconnect peer", logging.HostID("peer", pid), zap.Error(err))
|
|
|
|
}
|
|
|
|
}
|
2024-02-08 09:54:58 +00:00
|
|
|
_ = w.host.Peerstore().(wps.WakuPeerstore).SetScore(pid, snap.Score)
|
2023-06-07 20:18:37 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// SetHost sets the host to be able to mount or consume a protocol
|
2023-04-17 00:04:12 +00:00
|
|
|
func (w *WakuRelay) SetHost(h host.Host) {
|
|
|
|
w.host = h
|
|
|
|
}
|
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// Start initiates the WakuRelay protocol
|
2023-01-06 22:37:57 +00:00
|
|
|
func (w *WakuRelay) Start(ctx context.Context) error {
|
2023-09-13 05:18:44 +00:00
|
|
|
return w.CommonService.Start(ctx, w.start)
|
|
|
|
}
|
2023-02-08 14:20:40 +00:00
|
|
|
|
2023-09-13 05:18:44 +00:00
|
|
|
func (w *WakuRelay) start() error {
|
2023-10-20 19:56:18 +00:00
|
|
|
if w.bcaster == nil {
|
|
|
|
return errors.New("broadcaster not specified for relay")
|
|
|
|
}
|
2024-01-03 01:36:41 +00:00
|
|
|
ps, err := pubsub.NewGossipSub(w.Context(), w.host, w.relayParams.pubsubOpts...)
|
2021-04-28 20:10:44 +00:00
|
|
|
if err != nil {
|
2023-01-06 22:37:57 +00:00
|
|
|
return err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
w.pubsub = ps
|
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
err = w.CreateEventEmitters()
|
2023-09-19 06:05:29 +00:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2022-01-18 18:17:06 +00:00
|
|
|
w.log.Info("Relay protocol started")
|
2023-01-06 22:37:57 +00:00
|
|
|
return nil
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
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 {
|
2023-09-18 13:18:16 +00:00
|
|
|
defer w.topicsMutex.RUnlock()
|
|
|
|
w.topicsMutex.RLock()
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2021-11-19 16:19:48 +00:00
|
|
|
var result []string
|
2023-11-21 17:27:50 +00:00
|
|
|
for topic := range w.topics {
|
2021-04-28 20:10:44 +00:00
|
|
|
result = append(result, topic)
|
|
|
|
}
|
|
|
|
return result
|
|
|
|
}
|
|
|
|
|
2023-07-19 16:25:35 +00:00
|
|
|
// IsSubscribed indicates whether the node is subscribed to a pubsub topic or not
|
2023-05-04 14:04:54 +00:00
|
|
|
func (w *WakuRelay) IsSubscribed(topic string) bool {
|
2023-09-18 13:18:16 +00:00
|
|
|
w.topicsMutex.RLock()
|
|
|
|
defer w.topicsMutex.RUnlock()
|
2023-11-21 17:27:50 +00:00
|
|
|
_, ok := w.topics[topic]
|
2023-05-04 14:04:54 +00:00
|
|
|
return ok
|
|
|
|
}
|
|
|
|
|
2022-05-30 15:55:30 +00:00
|
|
|
// SetPubSub is used to set an implementation 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) {
|
2023-11-21 17:27:50 +00:00
|
|
|
topicData, ok := w.topics[topic]
|
2021-04-28 20:10:44 +00:00
|
|
|
if !ok { // Joins topic if node hasn't joined yet
|
2023-09-07 21:39:10 +00:00
|
|
|
err := w.pubsub.RegisterTopicValidator(topic, w.topicValidator(topic))
|
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to register topic validator", zap.String("pubsubTopic", topic), zap.Error(err))
|
2023-09-07 21:39:10 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
2021-04-28 20:10:44 +00:00
|
|
|
newTopic, err := w.pubsub.Join(string(topic))
|
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to join pubsubTopic", zap.String("pubsubTopic", topic), zap.Error(err))
|
2021-04-28 20:10:44 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
2023-06-07 20:18:37 +00:00
|
|
|
|
|
|
|
err = newTopic.SetScoreParams(w.topicParams)
|
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to set score params", zap.String("pubsubTopic", topic), zap.Error(err))
|
2023-06-07 20:18:37 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
w.topics[topic] = &pubsubTopicSubscriptionDetails{
|
|
|
|
topic: newTopic,
|
|
|
|
}
|
|
|
|
|
|
|
|
return newTopic, nil
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
return topicData.topic, nil
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
func (w *WakuRelay) subscribeToPubsubTopic(topic string) (*pubsubTopicSubscriptionDetails, error) {
|
|
|
|
w.topicsMutex.Lock()
|
|
|
|
defer w.topicsMutex.Unlock()
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Info("subscribing to underlying pubsubTopic", zap.String("pubsubTopic", topic))
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
result, ok := w.topics[topic]
|
2021-04-28 20:10:44 +00:00
|
|
|
if !ok {
|
|
|
|
pubSubTopic, err := w.upsertTopic(topic)
|
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to upsert topic", zap.String("pubsubTopic", topic), zap.Error(err))
|
2021-11-01 14:42:55 +00:00
|
|
|
return nil, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
subscription, err := pubSubTopic.Subscribe(pubsub.WithBufferSize(1024))
|
2021-04-28 20:10:44 +00:00
|
|
|
if err != nil {
|
2021-11-01 14:42:55 +00:00
|
|
|
return nil, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
2023-07-05 19:17:43 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
w.WaitGroup().Add(1)
|
2023-11-21 17:27:50 +00:00
|
|
|
go w.pubsubTopicMsgHandler(subscription)
|
2023-10-20 19:56:18 +00:00
|
|
|
|
2023-09-19 06:05:29 +00:00
|
|
|
evtHandler, err := w.addPeerTopicEventListener(pubSubTopic)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
w.topics[topic].contentSubs = make(map[int]*Subscription)
|
|
|
|
w.topics[topic].subscription = subscription
|
|
|
|
w.topics[topic].topicEventHandler = evtHandler
|
2023-07-05 19:17:43 +00:00
|
|
|
|
2023-09-27 06:46:37 +00:00
|
|
|
err = w.emitters.EvtRelaySubscribed.Emit(EvtRelaySubscribed{topic, pubSubTopic})
|
2023-07-05 19:17:43 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
w.log.Info("gossipsub subscription", zap.String("pubsubTopic", subscription.Topic()))
|
2024-01-03 05:41:11 +00:00
|
|
|
w.metrics.SetPubSubTopics(len(w.topics))
|
2023-11-21 17:27:50 +00:00
|
|
|
result = w.topics[topic]
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
return result, nil
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2024-01-03 15:14:59 +00:00
|
|
|
// Publish is used to broadcast a WakuMessage to a pubsub topic. The pubsubTopic is derived from contentTopic
|
2023-10-30 16:30:25 +00:00
|
|
|
// specified in the message via autosharding. To publish to a specific pubsubTopic, the `WithPubSubTopic` option should
|
|
|
|
// be provided
|
2024-05-03 16:07:03 +00:00
|
|
|
func (w *WakuRelay) Publish(ctx context.Context, message *pb.WakuMessage, opts ...PublishOption) (pb.MessageHash, error) {
|
2021-04-28 20:10:44 +00:00
|
|
|
// Publish a `WakuMessage` to a PubSub topic.
|
|
|
|
if w.pubsub == nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, errors.New("PubSub hasn't been set")
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if message == nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, errors.New("message can't be null")
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2023-10-30 16:30:25 +00:00
|
|
|
err := message.Validate()
|
|
|
|
if err != nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, err
|
2023-10-24 16:26:02 +00:00
|
|
|
}
|
|
|
|
|
2023-10-30 16:30:25 +00:00
|
|
|
params := new(publishParameters)
|
|
|
|
for _, opt := range opts {
|
|
|
|
opt(params)
|
|
|
|
}
|
|
|
|
|
|
|
|
if params.pubsubTopic == "" {
|
|
|
|
params.pubsubTopic, err = waku_proto.GetPubSubTopicFromContentTopic(message.ContentTopic)
|
|
|
|
if err != nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, err
|
2023-10-30 16:30:25 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if !w.EnoughPeersToPublishToTopic(params.pubsubTopic) {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, errors.New("not enough peers to publish")
|
2021-12-06 08:43:00 +00:00
|
|
|
}
|
|
|
|
|
2024-03-25 15:33:21 +00:00
|
|
|
w.topicsMutex.Lock()
|
|
|
|
defer w.topicsMutex.Unlock()
|
2023-11-21 17:27:50 +00:00
|
|
|
|
2023-10-30 16:30:25 +00:00
|
|
|
pubSubTopic, err := w.upsertTopic(params.pubsubTopic)
|
2021-04-28 20:10:44 +00:00
|
|
|
if err != nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
out, err := proto.Marshal(message)
|
|
|
|
if err != nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2024-01-03 01:36:41 +00:00
|
|
|
if len(out) > w.relayParams.maxMsgSizeBytes {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, errors.New("message size exceeds gossipsub max message size")
|
2023-12-01 01:23:28 +00:00
|
|
|
}
|
|
|
|
|
2021-04-28 20:10:44 +00:00
|
|
|
err = pubSubTopic.Publish(ctx, out)
|
|
|
|
if err != nil {
|
2024-05-03 16:07:03 +00:00
|
|
|
return pb.MessageHash{}, err
|
2021-04-28 20:10:44 +00:00
|
|
|
}
|
|
|
|
|
2023-10-30 16:30:25 +00:00
|
|
|
hash := message.Hash(params.pubsubTopic)
|
2021-04-28 20:10:44 +00:00
|
|
|
|
2024-05-03 16:07:03 +00:00
|
|
|
w.logMessages.Debug("waku.relay published", zap.String("pubsubTopic", params.pubsubTopic), logging.Hash(hash), zap.Int64("publishTime", w.timesource.Now().UnixNano()), zap.Int("payloadSizeBytes", len(message.Payload)))
|
2022-11-03 13:53:33 +00:00
|
|
|
|
2021-04-28 20:10:44 +00:00
|
|
|
return hash, nil
|
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
func (w *WakuRelay) getSubscription(contentFilter waku_proto.ContentFilter) (*Subscription, error) {
|
|
|
|
w.topicsMutex.RLock()
|
|
|
|
defer w.topicsMutex.RUnlock()
|
|
|
|
topicData, ok := w.topics[contentFilter.PubsubTopic]
|
|
|
|
if ok {
|
|
|
|
for _, sub := range topicData.contentSubs {
|
|
|
|
if sub.contentFilter.Equals(contentFilter) {
|
|
|
|
if sub.noConsume { //This check is to ensure that default no-consumer subscription is not returned
|
|
|
|
continue
|
|
|
|
}
|
|
|
|
return sub, nil
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
return nil, errors.New("no subscription found for content topic")
|
|
|
|
}
|
|
|
|
|
2023-11-07 14:56:48 +00:00
|
|
|
// GetSubscriptionWithPubsubTopic fetches subscription matching pubsub and contentTopic
|
|
|
|
func (w *WakuRelay) GetSubscriptionWithPubsubTopic(pubsubTopic string, contentTopic string) (*Subscription, error) {
|
|
|
|
var contentFilter waku_proto.ContentFilter
|
|
|
|
if contentTopic != "" {
|
|
|
|
contentFilter = waku_proto.NewContentFilter(pubsubTopic, contentTopic)
|
|
|
|
} else {
|
|
|
|
contentFilter = waku_proto.NewContentFilter(pubsubTopic)
|
|
|
|
}
|
2023-11-24 04:56:06 +00:00
|
|
|
sub, err := w.getSubscription(contentFilter)
|
|
|
|
if err != nil {
|
|
|
|
err = errors.New("no subscription found for pubsubTopic")
|
|
|
|
}
|
|
|
|
return sub, err
|
2023-11-07 14:56:48 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// GetSubscription fetches subscription matching a contentTopic(via autosharding)
|
2023-10-20 19:56:18 +00:00
|
|
|
func (w *WakuRelay) GetSubscription(contentTopic string) (*Subscription, error) {
|
2023-11-07 14:56:48 +00:00
|
|
|
pubsubTopic, err := waku_proto.GetPubSubTopicFromContentTopic(contentTopic)
|
2023-10-20 19:56:18 +00:00
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to derive pubsubTopic", zap.Error(err), zap.String("contentTopic", contentTopic))
|
2023-10-20 19:56:18 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
2023-11-07 14:56:48 +00:00
|
|
|
contentFilter := waku_proto.NewContentFilter(pubsubTopic, contentTopic)
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
return w.getSubscription(contentFilter)
|
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() {
|
2023-09-13 05:18:44 +00:00
|
|
|
w.CommonService.Stop(func() {
|
|
|
|
w.host.RemoveStreamHandler(WakuRelayID_v200)
|
|
|
|
w.emitters.EvtRelaySubscribed.Close()
|
|
|
|
w.emitters.EvtRelayUnsubscribed.Close()
|
|
|
|
})
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
|
|
|
|
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
|
|
|
|
}
|
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
// subscribe returns list of Subscription to receive messages based on content filter
|
|
|
|
func (w *WakuRelay) subscribe(ctx context.Context, contentFilter waku_proto.ContentFilter, opts ...RelaySubscribeOption) ([]*Subscription, error) {
|
|
|
|
|
|
|
|
var subscriptions []*Subscription
|
|
|
|
pubSubTopicMap, err := waku_proto.ContentFilterToPubSubTopicMap(contentFilter)
|
2021-11-01 14:42:55 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2023-10-20 19:56:18 +00:00
|
|
|
params := new(RelaySubscribeParameters)
|
2021-11-01 14:42:55 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
var optList []RelaySubscribeOption
|
|
|
|
optList = append(optList, opts...)
|
|
|
|
for _, opt := range optList {
|
|
|
|
err := opt(params)
|
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to apply option", zap.Error(err))
|
2023-10-20 19:56:18 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
2023-11-07 14:56:48 +00:00
|
|
|
if params.cacheSize <= 0 {
|
|
|
|
params.cacheSize = uint(DefaultRelaySubscriptionBufferSize)
|
|
|
|
}
|
2021-11-01 14:42:55 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
for pubSubTopic, cTopics := range pubSubTopicMap {
|
2023-11-21 17:27:50 +00:00
|
|
|
w.log.Info("subscribing to", zap.String("pubsubTopic", pubSubTopic), zap.Strings("contentTopics", cTopics))
|
2023-10-20 19:56:18 +00:00
|
|
|
var cFilter waku_proto.ContentFilter
|
|
|
|
cFilter.PubsubTopic = pubSubTopic
|
|
|
|
cFilter.ContentTopics = waku_proto.NewContentTopicSet(cTopics...)
|
2021-11-19 20:01:52 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
//Check if gossipsub subscription already exists for pubSubTopic
|
|
|
|
if !w.IsSubscribed(pubSubTopic) {
|
|
|
|
_, err := w.subscribeToPubsubTopic(cFilter.PubsubTopic)
|
|
|
|
if err != nil {
|
|
|
|
//TODO: Handle partial errors.
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to subscribe to pubsubTopic", zap.Error(err), zap.String("pubsubTopic", cFilter.PubsubTopic))
|
2023-10-20 19:56:18 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
}
|
2023-09-18 13:18:16 +00:00
|
|
|
|
2023-11-07 14:56:48 +00:00
|
|
|
subscription := w.bcaster.Register(cFilter, WithBufferSize(int(params.cacheSize)),
|
2023-10-24 20:41:42 +00:00
|
|
|
WithConsumerOption(params.dontConsume))
|
2023-10-20 19:56:18 +00:00
|
|
|
|
|
|
|
// Create Content subscription
|
2023-11-21 17:27:50 +00:00
|
|
|
w.topicsMutex.Lock()
|
|
|
|
topicData, ok := w.topics[pubSubTopic]
|
|
|
|
if ok {
|
|
|
|
topicData.contentSubs[subscription.ID] = subscription
|
2023-10-20 19:56:18 +00:00
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
w.topicsMutex.Unlock()
|
2023-10-20 19:56:18 +00:00
|
|
|
|
|
|
|
subscriptions = append(subscriptions, subscription)
|
|
|
|
go func() {
|
|
|
|
<-ctx.Done()
|
|
|
|
subscription.Unsubscribe()
|
|
|
|
}()
|
2021-11-06 10:49:47 +00:00
|
|
|
}
|
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
return subscriptions, nil
|
|
|
|
}
|
2021-11-06 10:49:47 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
// Subscribe returns a Subscription to receive messages as per contentFilter
|
|
|
|
// contentFilter can contain pubSubTopic and contentTopics or only contentTopics(in case of autosharding)
|
|
|
|
func (w *WakuRelay) Subscribe(ctx context.Context, contentFilter waku_proto.ContentFilter, opts ...RelaySubscribeOption) ([]*Subscription, error) {
|
|
|
|
return w.subscribe(ctx, contentFilter, opts...)
|
|
|
|
}
|
|
|
|
|
|
|
|
// Unsubscribe closes a subscription to a pubsub topic
|
|
|
|
func (w *WakuRelay) Unsubscribe(ctx context.Context, contentFilter waku_proto.ContentFilter) error {
|
2023-09-19 06:05:29 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
pubSubTopicMap, err := waku_proto.ContentFilterToPubSubTopicMap(contentFilter)
|
2021-11-06 10:49:47 +00:00
|
|
|
if err != nil {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("failed to derive pubsubTopic from contentFilter", zap.String("pubsubTopic", contentFilter.PubsubTopic),
|
|
|
|
zap.Strings("contentTopics", contentFilter.ContentTopicsList()))
|
2021-11-06 10:49:47 +00:00
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
w.topicsMutex.Lock()
|
|
|
|
defer w.topicsMutex.Unlock()
|
2023-09-07 21:39:10 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
for pubSubTopic, cTopics := range pubSubTopicMap {
|
|
|
|
cfTemp := waku_proto.NewContentFilter(pubSubTopic, cTopics...)
|
|
|
|
pubsubUnsubscribe := false
|
2023-11-21 17:27:50 +00:00
|
|
|
sub, ok := w.topics[pubSubTopic]
|
2023-10-20 19:56:18 +00:00
|
|
|
if !ok {
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Error("not subscribed to topic", zap.String("topic", pubSubTopic))
|
2023-10-20 19:56:18 +00:00
|
|
|
return errors.New("not subscribed to topic")
|
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
topicData, ok := w.topics[pubSubTopic]
|
|
|
|
if ok {
|
2023-10-20 19:56:18 +00:00
|
|
|
//Remove relevant subscription
|
2023-11-21 17:27:50 +00:00
|
|
|
for subID, sub := range topicData.contentSubs {
|
2023-10-20 19:56:18 +00:00
|
|
|
if sub.contentFilter.Equals(cfTemp) {
|
|
|
|
sub.Unsubscribe()
|
2023-11-21 17:27:50 +00:00
|
|
|
delete(topicData.contentSubs, subID)
|
2023-10-20 19:56:18 +00:00
|
|
|
}
|
|
|
|
}
|
2023-11-21 17:27:50 +00:00
|
|
|
|
|
|
|
if len(topicData.contentSubs) == 0 {
|
2023-10-20 19:56:18 +00:00
|
|
|
pubsubUnsubscribe = true
|
|
|
|
}
|
|
|
|
} else {
|
|
|
|
//Should not land here ideally
|
|
|
|
w.log.Error("pubsub subscriptions exists, but contentSubscription doesn't for contentFilter",
|
|
|
|
zap.String("pubsubTopic", pubSubTopic), zap.Strings("contentTopics", cTopics))
|
2023-07-05 19:17:43 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
return errors.New("unexpected error in unsubscribe")
|
|
|
|
}
|
2021-11-06 10:49:47 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
if pubsubUnsubscribe {
|
|
|
|
err = w.unsubscribeFromPubsubTopic(sub)
|
2021-11-01 14:42:55 +00:00
|
|
|
if err != nil {
|
2023-10-20 19:56:18 +00:00
|
|
|
return err
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
2024-01-03 05:41:11 +00:00
|
|
|
w.metrics.SetPubSubTopics(len(w.topics))
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
2023-10-20 19:56:18 +00:00
|
|
|
}
|
|
|
|
return nil
|
2021-11-01 14:42:55 +00:00
|
|
|
}
|
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
// unsubscribeFromPubsubTopic unsubscribes subscription from underlying pubsub.
|
|
|
|
// Note: caller has to acquire topicsMutex in order to avoid race conditions
|
2023-11-21 17:27:50 +00:00
|
|
|
func (w *WakuRelay) unsubscribeFromPubsubTopic(topicData *pubsubTopicSubscriptionDetails) error {
|
2023-02-08 19:20:42 +00:00
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
pubSubTopic := topicData.subscription.Topic()
|
2023-11-24 01:21:37 +00:00
|
|
|
w.log.Info("unsubscribing from pubsubTopic", zap.String("topic", pubSubTopic))
|
2021-11-01 14:42:55 +00:00
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
topicData.subscription.Cancel()
|
|
|
|
topicData.topicEventHandler.Cancel()
|
2023-08-16 01:40:00 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
w.bcaster.UnRegister(pubSubTopic)
|
2022-11-03 13:53:33 +00:00
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
err := topicData.topic.Close()
|
2023-10-20 19:56:18 +00:00
|
|
|
if err != nil {
|
2024-06-18 02:36:16 +00:00
|
|
|
w.log.Error("failed to close the pubsubTopic", zap.String("topic", pubSubTopic), zap.Error(err))
|
2023-10-20 19:56:18 +00:00
|
|
|
return err
|
|
|
|
}
|
2023-07-05 19:17:43 +00:00
|
|
|
|
2023-10-20 19:56:18 +00:00
|
|
|
w.RemoveTopicValidator(pubSubTopic)
|
2023-09-19 06:05:29 +00:00
|
|
|
|
2023-11-24 01:21:37 +00:00
|
|
|
err = w.pubsub.UnregisterTopicValidator(pubSubTopic)
|
|
|
|
if err != nil {
|
2024-06-18 02:36:16 +00:00
|
|
|
w.log.Error("failed to unregister topic validator", zap.String("topic", pubSubTopic), zap.Error(err))
|
2023-11-24 01:21:37 +00:00
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
delete(w.topics, pubSubTopic)
|
|
|
|
|
|
|
|
return w.emitters.EvtRelayUnsubscribed.Emit(EvtRelayUnsubscribed{pubSubTopic})
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
func (w *WakuRelay) pubsubTopicMsgHandler(sub *pubsub.Subscription) {
|
2023-09-19 06:05:29 +00:00
|
|
|
defer w.WaitGroup().Done()
|
2023-10-20 19:56:18 +00:00
|
|
|
|
2023-09-19 06:05:29 +00:00
|
|
|
for {
|
2023-10-20 19:56:18 +00:00
|
|
|
msg, err := sub.Next(w.Context())
|
2023-09-19 06:05:29 +00:00
|
|
|
if err != nil {
|
2023-10-20 19:56:18 +00:00
|
|
|
if !errors.Is(err, context.Canceled) {
|
|
|
|
w.log.Error("getting message from subscription", zap.Error(err))
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|
2023-10-20 19:56:18 +00:00
|
|
|
sub.Cancel()
|
|
|
|
return
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|
2023-10-24 16:26:02 +00:00
|
|
|
|
|
|
|
wakuMessage, err := pb.Unmarshal(msg.Data)
|
|
|
|
if err != nil {
|
2023-10-20 19:56:18 +00:00
|
|
|
w.log.Error("decoding message", zap.Error(err))
|
|
|
|
return
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|
2023-10-24 16:26:02 +00:00
|
|
|
|
2023-11-21 17:27:50 +00:00
|
|
|
envelope := waku_proto.NewEnvelope(wakuMessage, w.timesource.Now().UnixNano(), sub.Topic())
|
2023-10-20 19:56:18 +00:00
|
|
|
w.metrics.RecordMessage(envelope)
|
|
|
|
|
|
|
|
w.bcaster.Submit(envelope)
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|
2023-10-20 19:56:18 +00:00
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
// Params returns the gossipsub configuration parameters used by WakuRelay
|
|
|
|
func (w *WakuRelay) Params() pubsub.GossipSubParams {
|
|
|
|
return w.params
|
2023-09-19 06:05:29 +00:00
|
|
|
}
|