1
0
mirror of https://github.com/mainflux/mainflux.git synced 2025-04-29 13:49:28 +08:00
Aryan Godara d9c47045cd
MF-1630 - Replace old subscriptions with a new one instead of throwing an error (#1633)
* updated pubsub.subscribe() to delete and renew already subscribed objects

Signed-off-by: aryan <aryangodara03@gmail.com>

* fixing sync.mutex error

Signed-off-by: aryan <aryangodara03@gmail.com>

* made changes to pubsub_test.go to fix failing CI

Signed-off-by: aryan <aryangodara03@gmail.com>

* made some changes to pubsub.go

Signed-off-by: aryan <aryangodara03@gmail.com>

* made some changes to pubsub.go added flag var

Signed-off-by: aryan <aryangodara03@gmail.com>

* little code cleanup

Signed-off-by: aryan <aryangodara03@gmail.com>

* minor change, to make code more intuitive

Signed-off-by: aryan <aryangodara03@gmail.com>

* added comment, removed the ws_client file

Signed-off-by: aryan <aryangodara03@gmail.com>

* removed ErrAlreadySubscribed

Signed-off-by: aryan <aryangodara03@gmail.com>

* fixed some failing tests, will check again after semaphore result

Signed-off-by: aryan <aryangodara03@gmail.com>

* made similar changes for MQTT and rabbitMQ

Signed-off-by: aryan <aryangodara03@gmail.com>

* removed extra comment

Signed-off-by: aryan <aryangodara03@gmail.com>

* removed extra comment

Signed-off-by: aryan <aryangodara03@gmail.com>

* removed extra comment

Signed-off-by: aryan <aryangodara03@gmail.com>

* to fix failing CI

Signed-off-by: aryan <aryangodara03@gmail.com>

* checking code after adding conditions

Signed-off-by: aryan <aryangodara03@gmail.com>

* added ps.mu.Lock(), and changed len(s)

Signed-off-by: aryan <aryangodara03@gmail.com>

* added tests for failing unsubscribe

Signed-off-by: aryan <aryangodara03@gmail.com>

* shifted defer lock

Signed-off-by: aryan <aryangodara03@gmail.com>

* fixed mqtt/pubsub

Signed-off-by: aryan <aryangodara03@gmail.com>

* fix mqtt/pubsub typo

Signed-off-by: aryan <aryangodara03@gmail.com>

* add comment to improve readability

Signed-off-by: aryan <aryangodara03@gmail.com>

* add comment to better improve readability

Signed-off-by: aryan <aryangodara03@gmail.com>

* add comments for docs

Signed-off-by: aryan <aryangodara03@gmail.com>

* improve doc comment

Signed-off-by: aryan <aryangodara03@gmail.com>

* change to mqtt/pubsub

Signed-off-by: aryan <aryangodara03@gmail.com>

* remove unnecessary mutex lock

Signed-off-by: aryan <aryangodara03@gmail.com>

* change to pubsub

Signed-off-by: aryan <aryangodara03@gmail.com>

* add comments to improve readability

Signed-off-by: aryan <aryangodara03@gmail.com>

* fix typo in comment

Signed-off-by: aryan <aryangodara03@gmail.com>

Signed-off-by: aryan <aryangodara03@gmail.com>
2022-08-12 18:08:20 +02:00

168 lines
3.6 KiB
Go

// Copyright (c) Mainflux
// SPDX-License-Identifier: Apache-2.0
package nats
import (
"errors"
"fmt"
"sync"
"github.com/gogo/protobuf/proto"
log "github.com/mainflux/mainflux/logger"
"github.com/mainflux/mainflux/pkg/messaging"
broker "github.com/nats-io/nats.go"
)
const chansPrefix = "channels"
var (
ErrAlreadySubscribed = errors.New("already subscribed to topic")
ErrNotSubscribed = errors.New("not subscribed")
ErrEmptyTopic = errors.New("empty topic")
ErrEmptyID = errors.New("empty id")
ErrFailed = errors.New("failed")
)
var _ messaging.PubSub = (*pubsub)(nil)
type subscription struct {
*broker.Subscription
cancel func() error
}
type pubsub struct {
publisher
logger log.Logger
mu sync.Mutex
queue string
subscriptions map[string]map[string]subscription
}
// NewPubSub returns NATS message publisher/subscriber.
// Parameter queue specifies the queue for the Subscribe method.
// If queue is specified (is not an empty string), Subscribe method
// will execute NATS QueueSubscribe which is conceptually different
// from ordinary subscribe. For more information, please take a look
// here: https://docs.nats.io/developing-with-nats/receiving/queues.
// If the queue is empty, Subscribe will be used.
func NewPubSub(url, queue string, logger log.Logger) (messaging.PubSub, error) {
conn, err := broker.Connect(url)
if err != nil {
return nil, err
}
ret := &pubsub{
publisher: publisher{
conn: conn,
},
queue: queue,
logger: logger,
subscriptions: make(map[string]map[string]subscription),
}
return ret, nil
}
func (ps *pubsub) Subscribe(id, topic string, handler messaging.MessageHandler) error {
if id == "" {
return ErrEmptyID
}
if topic == "" {
return ErrEmptyTopic
}
ps.mu.Lock()
// Check topic
s, ok := ps.subscriptions[topic]
if ok {
// Check client ID
if _, ok := s[id]; ok {
// Unlocking, so that Unsubscribe() can access ps.subscriptions
ps.mu.Unlock()
if err := ps.Unsubscribe(id, topic); err != nil {
return err
}
ps.mu.Lock()
// value of s can be changed while ps.mu is unlocked
s = ps.subscriptions[topic]
}
}
defer ps.mu.Unlock()
if s == nil {
s = make(map[string]subscription)
ps.subscriptions[topic] = s
}
nh := ps.natsHandler(handler)
if ps.queue != "" {
sub, err := ps.conn.QueueSubscribe(topic, ps.queue, nh)
if err != nil {
return err
}
s[id] = subscription{
Subscription: sub,
cancel: handler.Cancel,
}
return nil
}
sub, err := ps.conn.Subscribe(topic, nh)
if err != nil {
return err
}
s[id] = subscription{
Subscription: sub,
cancel: handler.Cancel,
}
return nil
}
func (ps *pubsub) Unsubscribe(id, topic string) error {
if id == "" {
return ErrEmptyID
}
if topic == "" {
return ErrEmptyTopic
}
ps.mu.Lock()
defer ps.mu.Unlock()
// Check topic
s, ok := ps.subscriptions[topic]
if !ok {
return ErrNotSubscribed
}
// Check topic ID
current, ok := s[id]
if !ok {
return ErrNotSubscribed
}
if current.cancel != nil {
if err := current.cancel(); err != nil {
return err
}
}
if err := current.Unsubscribe(); err != nil {
return err
}
delete(s, id)
if len(s) == 0 {
delete(ps.subscriptions, topic)
}
return nil
}
func (ps *pubsub) natsHandler(h messaging.MessageHandler) broker.MsgHandler {
return func(m *broker.Msg) {
var msg messaging.Message
if err := proto.Unmarshal(m.Data, &msg); err != nil {
ps.logger.Warn(fmt.Sprintf("Failed to unmarshal received message: %s", err))
return
}
if err := h.Handle(msg); err != nil {
ps.logger.Warn(fmt.Sprintf("Failed to handle Mainflux message: %s", err))
}
}
}