Chris Lu
5 years ago
8 changed files with 205 additions and 35 deletions
-
4weed/command/msg_broker.go
-
2weed/messaging/broker/broker_append.go
-
2weed/messaging/broker/broker_grpc_server.go
-
51weed/messaging/broker/broker_grpc_server_publish.go
-
88weed/messaging/broker/broker_grpc_server_subscribe.go
-
4weed/messaging/broker/broker_server.go
-
80weed/messaging/broker/topic_lock.go
-
9weed/messaging/broker_grpc_server_subscribe.go
@ -1,4 +1,4 @@ |
|||
package messaging |
|||
package broker |
|||
|
|||
import ( |
|||
"context" |
@ -1,4 +1,4 @@ |
|||
package messaging |
|||
package broker |
|||
|
|||
import ( |
|||
"context" |
@ -0,0 +1,88 @@ |
|||
package broker |
|||
|
|||
import ( |
|||
"io" |
|||
"sync" |
|||
"time" |
|||
|
|||
"github.com/golang/protobuf/proto" |
|||
|
|||
"github.com/chrislusf/seaweedfs/weed/glog" |
|||
"github.com/chrislusf/seaweedfs/weed/pb/messaging_pb" |
|||
"github.com/chrislusf/seaweedfs/weed/util" |
|||
) |
|||
|
|||
func (broker *MessageBroker) Subscribe(stream messaging_pb.SeaweedMessaging_SubscribeServer) error { |
|||
|
|||
// process initial request
|
|||
in, err := stream.Recv() |
|||
if err == io.EOF { |
|||
return nil |
|||
} |
|||
if err != nil { |
|||
return err |
|||
} |
|||
|
|||
subscriberId := in.Init.SubscriberId |
|||
|
|||
// get lock
|
|||
tp := TopicPartition{ |
|||
Namespace: in.Init.Namespace, |
|||
Topic: in.Init.Topic, |
|||
Partition: in.Init.Partition, |
|||
} |
|||
lock := broker.topicLocks.RequestSubscriberLock(tp) |
|||
defer broker.topicLocks.ReleaseLock(tp, false) |
|||
cond := sync.NewCond(&lock.Mutex) |
|||
|
|||
lastReadTime := time.Now() |
|||
switch in.Init.StartPosition { |
|||
case messaging_pb.SubscriberMessage_InitMessage_TIMESTAMP: |
|||
lastReadTime = time.Unix(0, in.Init.TimestampNs) |
|||
case messaging_pb.SubscriberMessage_InitMessage_LATEST: |
|||
case messaging_pb.SubscriberMessage_InitMessage_EARLIEST: |
|||
} |
|||
|
|||
// how to process each message
|
|||
// an error returned will end the subscription
|
|||
eachMessageFn := func(m *messaging_pb.Message) error { |
|||
err := stream.Send(&messaging_pb.BrokerMessage{ |
|||
Data: m, |
|||
}) |
|||
if err != nil { |
|||
glog.V(0).Infof("=> subscriber %v: %+v", subscriberId, err) |
|||
} |
|||
return err |
|||
} |
|||
|
|||
// loop through all messages
|
|||
for { |
|||
|
|||
_, buf := lock.logBuffer.ReadFromBuffer(lastReadTime) |
|||
|
|||
for pos := 0; pos+4 < len(buf); { |
|||
|
|||
size := util.BytesToUint32(buf[pos : pos+4]) |
|||
entryData := buf[pos+4 : pos+4+int(size)] |
|||
|
|||
m := &messaging_pb.Message{} |
|||
if err = proto.Unmarshal(entryData, m); err != nil { |
|||
glog.Errorf("unexpected unmarshal messaging_pb.Message: %v", err) |
|||
pos += 4 + int(size) |
|||
continue |
|||
} |
|||
|
|||
if err = eachMessageFn(m); err != nil { |
|||
return err |
|||
} |
|||
|
|||
lastReadTime = time.Unix(0, m.Timestamp) |
|||
pos += 4 + int(size) |
|||
} |
|||
|
|||
lock.Mutex.Lock() |
|||
cond.Wait() |
|||
lock.Mutex.Unlock() |
|||
} |
|||
|
|||
} |
@ -0,0 +1,80 @@ |
|||
package broker |
|||
|
|||
import ( |
|||
"sync" |
|||
"time" |
|||
|
|||
"github.com/chrislusf/seaweedfs/weed/util/log_buffer" |
|||
) |
|||
|
|||
type TopicPartition struct { |
|||
Namespace string |
|||
Topic string |
|||
Partition int32 |
|||
} |
|||
type TopicLock struct { |
|||
sync.Mutex |
|||
subscriberCount int |
|||
publisherCount int |
|||
logBuffer *log_buffer.LogBuffer |
|||
} |
|||
|
|||
type TopicLocks struct { |
|||
sync.Mutex |
|||
locks map[TopicPartition]*TopicLock |
|||
} |
|||
|
|||
func NewTopicLocks() *TopicLocks { |
|||
return &TopicLocks{ |
|||
locks: make(map[TopicPartition]*TopicLock), |
|||
} |
|||
} |
|||
|
|||
func (tl *TopicLocks) RequestSubscriberLock(partition TopicPartition) *TopicLock { |
|||
tl.Lock() |
|||
defer tl.Unlock() |
|||
|
|||
lock, found := tl.locks[partition] |
|||
if !found { |
|||
lock = &TopicLock{} |
|||
tl.locks[partition] = lock |
|||
} |
|||
lock.subscriberCount++ |
|||
|
|||
return lock |
|||
} |
|||
|
|||
func (tl *TopicLocks) RequestPublisherLock(partition TopicPartition, flushFn func(startTime, stopTime time.Time, buf []byte)) *log_buffer.LogBuffer { |
|||
tl.Lock() |
|||
defer tl.Unlock() |
|||
|
|||
lock, found := tl.locks[partition] |
|||
if !found { |
|||
lock = &TopicLock{} |
|||
tl.locks[partition] = lock |
|||
} |
|||
lock.publisherCount++ |
|||
cond := sync.NewCond(&lock.Mutex) |
|||
lock.logBuffer = log_buffer.NewLogBuffer(time.Minute, flushFn, func() { |
|||
cond.Broadcast() |
|||
}) |
|||
return lock.logBuffer |
|||
} |
|||
|
|||
func (tl *TopicLocks) ReleaseLock(partition TopicPartition, isPublisher bool) { |
|||
tl.Lock() |
|||
defer tl.Unlock() |
|||
|
|||
lock, found := tl.locks[partition] |
|||
if !found { |
|||
return |
|||
} |
|||
if isPublisher { |
|||
lock.publisherCount-- |
|||
} else { |
|||
lock.subscriberCount-- |
|||
} |
|||
if lock.subscriberCount <= 0 && lock.publisherCount <= 0 { |
|||
delete(tl.locks, partition) |
|||
} |
|||
} |
@ -1,9 +0,0 @@ |
|||
package messaging |
|||
|
|||
import ( |
|||
"github.com/chrislusf/seaweedfs/weed/pb/messaging_pb" |
|||
) |
|||
|
|||
func (broker *MessageBroker) Subscribe(server messaging_pb.SeaweedMessaging_SubscribeServer) error { |
|||
panic("implement me") |
|||
} |
Write
Preview
Loading…
Cancel
Save
Reference in new issue