tendermint/libs/pubsub/pubsub.go
Alexander Bezobchuk ab0835463f abci: Refactor tagging events using list of lists (#3643)
## PR

This PR introduces a fundamental breaking change to the structure of ABCI response and tx tags and the way they're processed. Namely, the SDK can support more complex and aggregated events for distribution and slashing. In addition, block responses can include duplicate keys in events.

    Implement new Event type. An event has a type and a list of KV pairs (ie. list-of-lists). Typical events may look like:

"rewards": [{"amount": "5000uatom", "validator": "...", "recipient": "..."}]
"sender": [{"address": "...", "balance": "100uatom"}]

The events are indexed by {even.type}.{even.attribute[i].key}/.... In this case a client would subscribe or query for rewards.recipient='...'

    ABCI response types and related types now include Events []Event instead of Tags []cmn.KVPair.
    PubSub logic now publishes/matches against map[string][]string instead of map[string]string to support duplicate keys in response events (from #1385). A match is successful if the value is found in the slice of strings.

closes: #1859
closes: #2905

## Commits:

* Implement Event ABCI type and updates responses to use events

* Update messages_test.go

* Update kvstore.go

* Update event_bus.go

* Update subscription.go

* Update pubsub.go

* Update kvstore.go

* Update query logic to handle slice of strings in events

* Update Empty#Matches and unit tests

* Update pubsub logic

* Update EventBus#Publish

* Update kv tx indexer

* Update godocs

* Update ResultEvent to use slice of strings; update RPC

* Update more tests

* Update abci.md

* Check for key in validateAndStringifyEvents

* Fix KV indexer to skip empty keys

* Fix linting errors

* Update CHANGELOG_PENDING.md

* Update docs/spec/abci/abci.md

Co-Authored-By: Federico Kunze <31522760+fedekunze@users.noreply.github.com>

* Update abci/types/types.proto

Co-Authored-By: Ethan Buchman <ethan@coinculture.info>

* Update docs/spec/abci/abci.md

Co-Authored-By: Ethan Buchman <ethan@coinculture.info>

* Update libs/pubsub/query/query.go

Co-Authored-By: Ethan Buchman <ethan@coinculture.info>

* Update match function to match if ANY value matches

* Implement TestSubscribeDuplicateKeys

* Update TestMatches to include multi-key test cases

* Update events.go

* Update Query interface godoc

* Update match godoc

* Add godoc for matchValue

* DRY-up tx indexing

* Return error from PublishWithEvents in EventBus#Publish

* Update PublishEventNewBlockHeader to return an error

* Fix build

* Update events doc in ABCI

* Update ABCI events godoc

* Implement TestEventBusPublishEventTxDuplicateKeys

* Update TestSubscribeDuplicateKeys to be table-driven

* Remove mod file

* Remove markdown from events godoc

* Implement TestTxSearchDeprecatedIndexing test
2019-06-12 14:03:45 +02:00

420 lines
12 KiB
Go

// Package pubsub implements a pub-sub model with a single publisher (Server)
// and multiple subscribers (clients).
//
// Though you can have multiple publishers by sharing a pointer to a server or
// by giving the same channel to each publisher and publishing messages from
// that channel (fan-in).
//
// Clients subscribe for messages, which could be of any type, using a query.
// When some message is published, we match it with all queries. If there is a
// match, this message will be pushed to all clients, subscribed to that query.
// See query subpackage for our implementation.
//
// Example:
//
// q, err := query.New("account.name='John'")
// if err != nil {
// return err
// }
// ctx, cancel := context.WithTimeout(context.Background(), 1 * time.Second)
// defer cancel()
// subscription, err := pubsub.Subscribe(ctx, "johns-transactions", q)
// if err != nil {
// return err
// }
//
// for {
// select {
// case msg <- subscription.Out():
// // handle msg.Data() and msg.Events()
// case <-subscription.Cancelled():
// return subscription.Err()
// }
// }
//
package pubsub
import (
"context"
"errors"
"sync"
cmn "github.com/tendermint/tendermint/libs/common"
)
type operation int
const (
sub operation = iota
pub
unsub
shutdown
)
var (
// ErrSubscriptionNotFound is returned when a client tries to unsubscribe
// from not existing subscription.
ErrSubscriptionNotFound = errors.New("subscription not found")
// ErrAlreadySubscribed is returned when a client tries to subscribe twice or
// more using the same query.
ErrAlreadySubscribed = errors.New("already subscribed")
)
// Query defines an interface for a query to be used for subscribing. A query
// matches against a map of events. Each key in this map is a composite of the
// even type and an attribute key (e.g. "{eventType}.{eventAttrKey}") and the
// values are the event values that are contained under that relationship. This
// allows event types to repeat themselves with the same set of keys and
// different values.
type Query interface {
Matches(events map[string][]string) bool
String() string
}
type cmd struct {
op operation
// subscribe, unsubscribe
query Query
subscription *Subscription
clientID string
// publish
msg interface{}
events map[string][]string
}
// Server allows clients to subscribe/unsubscribe for messages, publishing
// messages with or without events, and manages internal state.
type Server struct {
cmn.BaseService
cmds chan cmd
cmdsCap int
// check if we have subscription before
// subscribing or unsubscribing
mtx sync.RWMutex
subscriptions map[string]map[string]struct{} // subscriber -> query (string) -> empty struct
}
// Option sets a parameter for the server.
type Option func(*Server)
// NewServer returns a new server. See the commentary on the Option functions
// for a detailed description of how to configure buffering. If no options are
// provided, the resulting server's queue is unbuffered.
func NewServer(options ...Option) *Server {
s := &Server{
subscriptions: make(map[string]map[string]struct{}),
}
s.BaseService = *cmn.NewBaseService(nil, "PubSub", s)
for _, option := range options {
option(s)
}
// if BufferCapacity option was not set, the channel is unbuffered
s.cmds = make(chan cmd, s.cmdsCap)
return s
}
// BufferCapacity allows you to specify capacity for the internal server's
// queue. Since the server, given Y subscribers, could only process X messages,
// this option could be used to survive spikes (e.g. high amount of
// transactions during peak hours).
func BufferCapacity(cap int) Option {
return func(s *Server) {
if cap > 0 {
s.cmdsCap = cap
}
}
}
// BufferCapacity returns capacity of the internal server's queue.
func (s *Server) BufferCapacity() int {
return s.cmdsCap
}
// Subscribe creates a subscription for the given client.
//
// An error will be returned to the caller if the context is canceled or if
// subscription already exist for pair clientID and query.
//
// outCapacity can be used to set a capacity for Subscription#Out channel (1 by
// default). Panics if outCapacity is less than or equal to zero. If you want
// an unbuffered channel, use SubscribeUnbuffered.
func (s *Server) Subscribe(ctx context.Context, clientID string, query Query, outCapacity ...int) (*Subscription, error) {
outCap := 1
if len(outCapacity) > 0 {
if outCapacity[0] <= 0 {
panic("Negative or zero capacity. Use SubscribeUnbuffered if you want an unbuffered channel")
}
outCap = outCapacity[0]
}
return s.subscribe(ctx, clientID, query, outCap)
}
// SubscribeUnbuffered does the same as Subscribe, except it returns a
// subscription with unbuffered channel. Use with caution as it can freeze the
// server.
func (s *Server) SubscribeUnbuffered(ctx context.Context, clientID string, query Query) (*Subscription, error) {
return s.subscribe(ctx, clientID, query, 0)
}
func (s *Server) subscribe(ctx context.Context, clientID string, query Query, outCapacity int) (*Subscription, error) {
s.mtx.RLock()
clientSubscriptions, ok := s.subscriptions[clientID]
if ok {
_, ok = clientSubscriptions[query.String()]
}
s.mtx.RUnlock()
if ok {
return nil, ErrAlreadySubscribed
}
subscription := NewSubscription(outCapacity)
select {
case s.cmds <- cmd{op: sub, clientID: clientID, query: query, subscription: subscription}:
s.mtx.Lock()
if _, ok = s.subscriptions[clientID]; !ok {
s.subscriptions[clientID] = make(map[string]struct{})
}
s.subscriptions[clientID][query.String()] = struct{}{}
s.mtx.Unlock()
return subscription, nil
case <-ctx.Done():
return nil, ctx.Err()
case <-s.Quit():
return nil, nil
}
}
// Unsubscribe removes the subscription on the given query. An error will be
// returned to the caller if the context is canceled or if subscription does
// not exist.
func (s *Server) Unsubscribe(ctx context.Context, clientID string, query Query) error {
s.mtx.RLock()
clientSubscriptions, ok := s.subscriptions[clientID]
if ok {
_, ok = clientSubscriptions[query.String()]
}
s.mtx.RUnlock()
if !ok {
return ErrSubscriptionNotFound
}
select {
case s.cmds <- cmd{op: unsub, clientID: clientID, query: query}:
s.mtx.Lock()
delete(clientSubscriptions, query.String())
if len(clientSubscriptions) == 0 {
delete(s.subscriptions, clientID)
}
s.mtx.Unlock()
return nil
case <-ctx.Done():
return ctx.Err()
case <-s.Quit():
return nil
}
}
// UnsubscribeAll removes all client subscriptions. An error will be returned
// to the caller if the context is canceled or if subscription does not exist.
func (s *Server) UnsubscribeAll(ctx context.Context, clientID string) error {
s.mtx.RLock()
_, ok := s.subscriptions[clientID]
s.mtx.RUnlock()
if !ok {
return ErrSubscriptionNotFound
}
select {
case s.cmds <- cmd{op: unsub, clientID: clientID}:
s.mtx.Lock()
delete(s.subscriptions, clientID)
s.mtx.Unlock()
return nil
case <-ctx.Done():
return ctx.Err()
case <-s.Quit():
return nil
}
}
// NumClients returns the number of clients.
func (s *Server) NumClients() int {
s.mtx.RLock()
defer s.mtx.RUnlock()
return len(s.subscriptions)
}
// NumClientSubscriptions returns the number of subscriptions the client has.
func (s *Server) NumClientSubscriptions(clientID string) int {
s.mtx.RLock()
defer s.mtx.RUnlock()
return len(s.subscriptions[clientID])
}
// Publish publishes the given message. An error will be returned to the caller
// if the context is canceled.
func (s *Server) Publish(ctx context.Context, msg interface{}) error {
return s.PublishWithEvents(ctx, msg, make(map[string][]string))
}
// PublishWithEvents publishes the given message with the set of events. The set
// is matched with clients queries. If there is a match, the message is sent to
// the client.
func (s *Server) PublishWithEvents(ctx context.Context, msg interface{}, events map[string][]string) error {
select {
case s.cmds <- cmd{op: pub, msg: msg, events: events}:
return nil
case <-ctx.Done():
return ctx.Err()
case <-s.Quit():
return nil
}
}
// OnStop implements Service.OnStop by shutting down the server.
func (s *Server) OnStop() {
s.cmds <- cmd{op: shutdown}
}
// NOTE: not goroutine safe
type state struct {
// query string -> client -> subscription
subscriptions map[string]map[string]*Subscription
// query string -> queryPlusRefCount
queries map[string]*queryPlusRefCount
}
// queryPlusRefCount holds a pointer to a query and reference counter. When
// refCount is zero, query will be removed.
type queryPlusRefCount struct {
q Query
refCount int
}
// OnStart implements Service.OnStart by starting the server.
func (s *Server) OnStart() error {
go s.loop(state{
subscriptions: make(map[string]map[string]*Subscription),
queries: make(map[string]*queryPlusRefCount),
})
return nil
}
// OnReset implements Service.OnReset
func (s *Server) OnReset() error {
return nil
}
func (s *Server) loop(state state) {
loop:
for cmd := range s.cmds {
switch cmd.op {
case unsub:
if cmd.query != nil {
state.remove(cmd.clientID, cmd.query.String(), ErrUnsubscribed)
} else {
state.removeClient(cmd.clientID, ErrUnsubscribed)
}
case shutdown:
state.removeAll(nil)
break loop
case sub:
state.add(cmd.clientID, cmd.query, cmd.subscription)
case pub:
state.send(cmd.msg, cmd.events)
}
}
}
func (state *state) add(clientID string, q Query, subscription *Subscription) {
qStr := q.String()
// initialize subscription for this client per query if needed
if _, ok := state.subscriptions[qStr]; !ok {
state.subscriptions[qStr] = make(map[string]*Subscription)
}
// create subscription
state.subscriptions[qStr][clientID] = subscription
// initialize query if needed
if _, ok := state.queries[qStr]; !ok {
state.queries[qStr] = &queryPlusRefCount{q: q, refCount: 0}
}
// increment reference counter
state.queries[qStr].refCount++
}
func (state *state) remove(clientID string, qStr string, reason error) {
clientSubscriptions, ok := state.subscriptions[qStr]
if !ok {
return
}
subscription, ok := clientSubscriptions[clientID]
if !ok {
return
}
subscription.cancel(reason)
// remove client from query map.
// if query has no other clients subscribed, remove it.
delete(state.subscriptions[qStr], clientID)
if len(state.subscriptions[qStr]) == 0 {
delete(state.subscriptions, qStr)
}
// decrease ref counter in queries
state.queries[qStr].refCount--
// remove the query if nobody else is using it
if state.queries[qStr].refCount == 0 {
delete(state.queries, qStr)
}
}
func (state *state) removeClient(clientID string, reason error) {
for qStr, clientSubscriptions := range state.subscriptions {
if _, ok := clientSubscriptions[clientID]; ok {
state.remove(clientID, qStr, reason)
}
}
}
func (state *state) removeAll(reason error) {
for qStr, clientSubscriptions := range state.subscriptions {
for clientID := range clientSubscriptions {
state.remove(clientID, qStr, reason)
}
}
}
func (state *state) send(msg interface{}, events map[string][]string) {
for qStr, clientSubscriptions := range state.subscriptions {
q := state.queries[qStr].q
if q.Matches(events) {
for clientID, subscription := range clientSubscriptions {
if cap(subscription.out) == 0 {
// block on unbuffered channel
subscription.out <- NewMessage(msg, events)
} else {
// don't block on buffered channels
select {
case subscription.out <- NewMessage(msg, events):
default:
state.remove(clientID, qStr, ErrOutOfCapacity)
}
}
}
}
}
}