Merge branch 'master' into micro
This commit is contained in:
commit
24e6840ea4
@ -4,28 +4,35 @@ type Broker interface {
|
|||||||
Address() string
|
Address() string
|
||||||
Connect() error
|
Connect() error
|
||||||
Disconnect() error
|
Disconnect() error
|
||||||
Init() error
|
Init(...Option) error
|
||||||
Publish(string, *Message) error
|
Publish(string, *Message, ...PublishOption) error
|
||||||
Subscribe(string, Handler) (Subscriber, error)
|
Subscribe(string, Handler, ...SubscribeOption) (Subscriber, error)
|
||||||
String() string
|
String() string
|
||||||
}
|
}
|
||||||
|
|
||||||
type Handler func(*Message)
|
// Handler is used to process messages via a subscription of a topic.
|
||||||
|
// The handler is passed a publication interface which contains the
|
||||||
|
// message and optional Ack method to acknowledge receipt of the message.
|
||||||
|
type Handler func(Publication) error
|
||||||
|
|
||||||
type Message struct {
|
type Message struct {
|
||||||
Header map[string]string
|
Header map[string]string
|
||||||
Body []byte
|
Body []byte
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// Publication is given to a subscription handler for processing
|
||||||
|
type Publication interface {
|
||||||
|
Topic() string
|
||||||
|
Message() *Message
|
||||||
|
Ack() error
|
||||||
|
}
|
||||||
|
|
||||||
type Subscriber interface {
|
type Subscriber interface {
|
||||||
|
Config() SubscribeOptions
|
||||||
Topic() string
|
Topic() string
|
||||||
Unsubscribe() error
|
Unsubscribe() error
|
||||||
}
|
}
|
||||||
|
|
||||||
type options struct{}
|
|
||||||
|
|
||||||
type Option func(*options)
|
|
||||||
|
|
||||||
var (
|
var (
|
||||||
DefaultBroker Broker = newHttpBroker([]string{})
|
DefaultBroker Broker = newHttpBroker([]string{})
|
||||||
)
|
)
|
||||||
@ -34,8 +41,8 @@ func NewBroker(addrs []string, opt ...Option) Broker {
|
|||||||
return newHttpBroker(addrs, opt...)
|
return newHttpBroker(addrs, opt...)
|
||||||
}
|
}
|
||||||
|
|
||||||
func Init() error {
|
func Init(opts ...Option) error {
|
||||||
return DefaultBroker.Init()
|
return DefaultBroker.Init(opts...)
|
||||||
}
|
}
|
||||||
|
|
||||||
func Connect() error {
|
func Connect() error {
|
||||||
@ -46,12 +53,12 @@ func Disconnect() error {
|
|||||||
return DefaultBroker.Disconnect()
|
return DefaultBroker.Disconnect()
|
||||||
}
|
}
|
||||||
|
|
||||||
func Publish(topic string, msg *Message) error {
|
func Publish(topic string, msg *Message, opts ...PublishOption) error {
|
||||||
return DefaultBroker.Publish(topic, msg)
|
return DefaultBroker.Publish(topic, msg, opts...)
|
||||||
}
|
}
|
||||||
|
|
||||||
func Subscribe(topic string, handler Handler) (Subscriber, error) {
|
func Subscribe(topic string, handler Handler, opts ...SubscribeOption) (Subscriber, error) {
|
||||||
return DefaultBroker.Subscribe(topic, handler)
|
return DefaultBroker.Subscribe(topic, handler, opts...)
|
||||||
}
|
}
|
||||||
|
|
||||||
func String() string {
|
func String() string {
|
||||||
|
@ -4,12 +4,15 @@ import (
|
|||||||
"bytes"
|
"bytes"
|
||||||
"encoding/json"
|
"encoding/json"
|
||||||
"fmt"
|
"fmt"
|
||||||
|
"io"
|
||||||
"io/ioutil"
|
"io/ioutil"
|
||||||
|
"math/rand"
|
||||||
"net"
|
"net"
|
||||||
"net/http"
|
"net/http"
|
||||||
"strconv"
|
"strconv"
|
||||||
"strings"
|
"strings"
|
||||||
"sync"
|
"sync"
|
||||||
|
"time"
|
||||||
|
|
||||||
log "github.com/golang/glog"
|
log "github.com/golang/glog"
|
||||||
"github.com/micro/go-micro/errors"
|
"github.com/micro/go-micro/errors"
|
||||||
@ -17,6 +20,10 @@ import (
|
|||||||
"github.com/pborman/uuid"
|
"github.com/pborman/uuid"
|
||||||
)
|
)
|
||||||
|
|
||||||
|
// HTTP Broker is a placeholder for actual message brokers.
|
||||||
|
// This should not really be used in production but useful
|
||||||
|
// in developer where you want zero dependencies.
|
||||||
|
|
||||||
type httpBroker struct {
|
type httpBroker struct {
|
||||||
id string
|
id string
|
||||||
address string
|
address string
|
||||||
@ -29,6 +36,7 @@ type httpBroker struct {
|
|||||||
}
|
}
|
||||||
|
|
||||||
type httpSubscriber struct {
|
type httpSubscriber struct {
|
||||||
|
opts SubscribeOptions
|
||||||
id string
|
id string
|
||||||
topic string
|
topic string
|
||||||
ch chan *httpSubscriber
|
ch chan *httpSubscriber
|
||||||
@ -36,10 +44,20 @@ type httpSubscriber struct {
|
|||||||
svc *registry.Service
|
svc *registry.Service
|
||||||
}
|
}
|
||||||
|
|
||||||
|
type httpPublication struct {
|
||||||
|
m *Message
|
||||||
|
t string
|
||||||
|
}
|
||||||
|
|
||||||
var (
|
var (
|
||||||
DefaultSubPath = "/_sub"
|
DefaultSubPath = "/_sub"
|
||||||
|
broadcastVersion = "ff.http.broadcast"
|
||||||
)
|
)
|
||||||
|
|
||||||
|
func init() {
|
||||||
|
rand.Seed(time.Now().Unix())
|
||||||
|
}
|
||||||
|
|
||||||
func newHttpBroker(addrs []string, opt ...Option) Broker {
|
func newHttpBroker(addrs []string, opt ...Option) Broker {
|
||||||
addr := ":0"
|
addr := ":0"
|
||||||
if len(addrs) > 0 && len(addrs[0]) > 0 {
|
if len(addrs) > 0 && len(addrs[0]) > 0 {
|
||||||
@ -55,6 +73,22 @@ func newHttpBroker(addrs []string, opt ...Option) Broker {
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
func (h *httpPublication) Ack() error {
|
||||||
|
return nil
|
||||||
|
}
|
||||||
|
|
||||||
|
func (h *httpPublication) Message() *Message {
|
||||||
|
return h.m
|
||||||
|
}
|
||||||
|
|
||||||
|
func (h *httpPublication) Topic() string {
|
||||||
|
return h.t
|
||||||
|
}
|
||||||
|
|
||||||
|
func (h *httpSubscriber) Config() SubscribeOptions {
|
||||||
|
return h.opts
|
||||||
|
}
|
||||||
|
|
||||||
func (h *httpSubscriber) Topic() string {
|
func (h *httpSubscriber) Topic() string {
|
||||||
return h.topic
|
return h.topic
|
||||||
}
|
}
|
||||||
@ -150,9 +184,10 @@ func (h *httpBroker) ServeHTTP(w http.ResponseWriter, req *http.Request) {
|
|||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
|
p := &httpPublication{m: m, t: topic}
|
||||||
h.RLock()
|
h.RLock()
|
||||||
for _, subscriber := range h.subscribers[topic] {
|
for _, subscriber := range h.subscribers[topic] {
|
||||||
subscriber.fn(m)
|
subscriber.fn(p)
|
||||||
}
|
}
|
||||||
h.RUnlock()
|
h.RUnlock()
|
||||||
}
|
}
|
||||||
@ -169,7 +204,7 @@ func (h *httpBroker) Disconnect() error {
|
|||||||
return h.stop()
|
return h.stop()
|
||||||
}
|
}
|
||||||
|
|
||||||
func (h *httpBroker) Init() error {
|
func (h *httpBroker) Init(opts ...Option) error {
|
||||||
if len(h.id) == 0 {
|
if len(h.id) == 0 {
|
||||||
h.id = "broker-" + uuid.NewUUID().String()
|
h.id = "broker-" + uuid.NewUUID().String()
|
||||||
}
|
}
|
||||||
@ -178,7 +213,7 @@ func (h *httpBroker) Init() error {
|
|||||||
return nil
|
return nil
|
||||||
}
|
}
|
||||||
|
|
||||||
func (h *httpBroker) Publish(topic string, msg *Message) error {
|
func (h *httpBroker) Publish(topic string, msg *Message, opts ...PublishOption) error {
|
||||||
s, err := registry.GetService("topic:" + topic)
|
s, err := registry.GetService("topic:" + topic)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return err
|
return err
|
||||||
@ -190,37 +225,70 @@ func (h *httpBroker) Publish(topic string, msg *Message) error {
|
|||||||
return err
|
return err
|
||||||
}
|
}
|
||||||
|
|
||||||
for _, service := range s {
|
fn := func(node *registry.Node, b io.Reader) {
|
||||||
for _, node := range service.Nodes {
|
r, err := http.Post(fmt.Sprintf("http://%s:%d%s", node.Address, node.Port, DefaultSubPath), "application/json", b)
|
||||||
r, err := http.Post(fmt.Sprintf("http://%s:%d%s", node.Address, node.Port, DefaultSubPath), "application/json", bytes.NewBuffer(b))
|
|
||||||
if err == nil {
|
if err == nil {
|
||||||
r.Body.Close()
|
r.Body.Close()
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
buf := bytes.NewBuffer(nil)
|
||||||
|
|
||||||
|
for _, service := range s {
|
||||||
|
// broadcast version means broadcast to all nodes
|
||||||
|
if service.Version == broadcastVersion {
|
||||||
|
for _, node := range service.Nodes {
|
||||||
|
buf.Reset()
|
||||||
|
buf.Write(b)
|
||||||
|
fn(node, buf)
|
||||||
}
|
}
|
||||||
return nil
|
return nil
|
||||||
}
|
}
|
||||||
|
|
||||||
func (h *httpBroker) Subscribe(topic string, handler Handler) (Subscriber, error) {
|
node := service.Nodes[rand.Int()%len(service.Nodes)]
|
||||||
|
buf.Reset()
|
||||||
|
buf.Write(b)
|
||||||
|
fn(node, buf)
|
||||||
|
return nil
|
||||||
|
}
|
||||||
|
|
||||||
|
buf.Reset()
|
||||||
|
buf = nil
|
||||||
|
|
||||||
|
return nil
|
||||||
|
}
|
||||||
|
|
||||||
|
func (h *httpBroker) Subscribe(topic string, handler Handler, opts ...SubscribeOption) (Subscriber, error) {
|
||||||
|
opt := newSubscribeOptions(opts...)
|
||||||
|
|
||||||
// parse address for host, port
|
// parse address for host, port
|
||||||
parts := strings.Split(h.Address(), ":")
|
parts := strings.Split(h.Address(), ":")
|
||||||
host := strings.Join(parts[:len(parts)-1], ":")
|
host := strings.Join(parts[:len(parts)-1], ":")
|
||||||
port, _ := strconv.Atoi(parts[len(parts)-1])
|
port, _ := strconv.Atoi(parts[len(parts)-1])
|
||||||
|
|
||||||
|
id := uuid.NewUUID().String()
|
||||||
|
|
||||||
// register service
|
// register service
|
||||||
node := ®istry.Node{
|
node := ®istry.Node{
|
||||||
Id: h.id,
|
Id: topic + "." + h.id + "." + id,
|
||||||
Address: host,
|
Address: host,
|
||||||
Port: port,
|
Port: port,
|
||||||
}
|
}
|
||||||
|
|
||||||
|
version := opt.Queue
|
||||||
|
if len(version) == 0 {
|
||||||
|
version = broadcastVersion
|
||||||
|
}
|
||||||
|
|
||||||
service := ®istry.Service{
|
service := ®istry.Service{
|
||||||
Name: "topic:" + topic,
|
Name: "topic:" + topic,
|
||||||
|
Version: version,
|
||||||
Nodes: []*registry.Node{node},
|
Nodes: []*registry.Node{node},
|
||||||
}
|
}
|
||||||
|
|
||||||
subscriber := &httpSubscriber{
|
subscriber := &httpSubscriber{
|
||||||
id: uuid.NewUUID().String(),
|
opts: opt,
|
||||||
|
id: id,
|
||||||
topic: topic,
|
topic: topic,
|
||||||
ch: h.unsubscribe,
|
ch: h.unsubscribe,
|
||||||
fn: handler,
|
fn: handler,
|
||||||
@ -234,7 +302,6 @@ func (h *httpBroker) Subscribe(topic string, handler Handler) (Subscriber, error
|
|||||||
h.Lock()
|
h.Lock()
|
||||||
h.subscribers[topic] = append(h.subscribers[topic], subscriber)
|
h.subscribers[topic] = append(h.subscribers[topic], subscriber)
|
||||||
h.Unlock()
|
h.Unlock()
|
||||||
|
|
||||||
return subscriber, nil
|
return subscriber, nil
|
||||||
}
|
}
|
||||||
|
|
||||||
|
48
broker/options.go
Normal file
48
broker/options.go
Normal file
@ -0,0 +1,48 @@
|
|||||||
|
package broker
|
||||||
|
|
||||||
|
type Options struct{}
|
||||||
|
|
||||||
|
type PublishOptions struct{}
|
||||||
|
|
||||||
|
type SubscribeOptions struct {
|
||||||
|
// AutoAck defaults to true. When a handler returns
|
||||||
|
// with a nil error the message is acked.
|
||||||
|
AutoAck bool
|
||||||
|
// Subscribers with the same queue name
|
||||||
|
// will create a shared subscription where each
|
||||||
|
// receives a subset of messages.
|
||||||
|
Queue string
|
||||||
|
}
|
||||||
|
|
||||||
|
type Option func(*Options)
|
||||||
|
|
||||||
|
type PublishOption func(*PublishOptions)
|
||||||
|
|
||||||
|
type SubscribeOption func(*SubscribeOptions)
|
||||||
|
|
||||||
|
// DisableAutoAck will disable auto acking of messages
|
||||||
|
// after they have been handled.
|
||||||
|
func DisableAutoAck() SubscribeOption {
|
||||||
|
return func(o *SubscribeOptions) {
|
||||||
|
o.AutoAck = false
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// QueueName sets the name of the queue to share messages on
|
||||||
|
func QueueName(name string) SubscribeOption {
|
||||||
|
return func(o *SubscribeOptions) {
|
||||||
|
o.Queue = name
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
func newSubscribeOptions(opts ...SubscribeOption) SubscribeOptions {
|
||||||
|
opt := SubscribeOptions{
|
||||||
|
AutoAck: true,
|
||||||
|
}
|
||||||
|
|
||||||
|
for _, o := range opts {
|
||||||
|
o(&opt)
|
||||||
|
}
|
||||||
|
|
||||||
|
return opt
|
||||||
|
}
|
@ -136,9 +136,12 @@ func (r *rpcClient) stream(ctx context.Context, address string, req Request) (St
|
|||||||
return nil, errors.InternalServerError("go.micro.client", fmt.Sprintf("Error sending request: %v", err))
|
return nil, errors.InternalServerError("go.micro.client", fmt.Sprintf("Error sending request: %v", err))
|
||||||
}
|
}
|
||||||
|
|
||||||
|
var once sync.Once
|
||||||
stream := &rpcStream{
|
stream := &rpcStream{
|
||||||
context: ctx,
|
context: ctx,
|
||||||
request: req,
|
request: req,
|
||||||
|
once: once,
|
||||||
|
closed: make(chan bool),
|
||||||
codec: newRpcPlusCodec(msg, c, cf),
|
codec: newRpcPlusCodec(msg, c, cf),
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -13,13 +13,25 @@ import (
|
|||||||
type rpcStream struct {
|
type rpcStream struct {
|
||||||
sync.RWMutex
|
sync.RWMutex
|
||||||
seq uint64
|
seq uint64
|
||||||
closed bool
|
once sync.Once
|
||||||
|
closed chan bool
|
||||||
err error
|
err error
|
||||||
request Request
|
request Request
|
||||||
codec clientCodec
|
codec clientCodec
|
||||||
context context.Context
|
context context.Context
|
||||||
}
|
}
|
||||||
|
|
||||||
|
func (r *rpcStream) isClosed() bool {
|
||||||
|
select {
|
||||||
|
case _, ok := <-r.closed:
|
||||||
|
if !ok {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
default:
|
||||||
|
}
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
|
||||||
func (r *rpcStream) Context() context.Context {
|
func (r *rpcStream) Context() context.Context {
|
||||||
return r.context
|
return r.context
|
||||||
}
|
}
|
||||||
@ -32,7 +44,7 @@ func (r *rpcStream) Send(msg interface{}) error {
|
|||||||
r.Lock()
|
r.Lock()
|
||||||
defer r.Unlock()
|
defer r.Unlock()
|
||||||
|
|
||||||
if r.closed {
|
if r.isClosed() {
|
||||||
r.err = errShutdown
|
r.err = errShutdown
|
||||||
return errShutdown
|
return errShutdown
|
||||||
}
|
}
|
||||||
@ -57,14 +69,14 @@ func (r *rpcStream) Recv(msg interface{}) error {
|
|||||||
r.Lock()
|
r.Lock()
|
||||||
defer r.Unlock()
|
defer r.Unlock()
|
||||||
|
|
||||||
if r.closed {
|
if r.isClosed() {
|
||||||
r.err = errShutdown
|
r.err = errShutdown
|
||||||
return errShutdown
|
return errShutdown
|
||||||
}
|
}
|
||||||
|
|
||||||
var resp response
|
var resp response
|
||||||
if err := r.codec.ReadResponseHeader(&resp); err != nil {
|
if err := r.codec.ReadResponseHeader(&resp); err != nil {
|
||||||
if err == io.EOF && !r.closed {
|
if err == io.EOF && !r.isClosed() {
|
||||||
r.err = io.ErrUnexpectedEOF
|
r.err = io.ErrUnexpectedEOF
|
||||||
return io.ErrUnexpectedEOF
|
return io.ErrUnexpectedEOF
|
||||||
}
|
}
|
||||||
@ -91,7 +103,7 @@ func (r *rpcStream) Recv(msg interface{}) error {
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
if r.err != nil && r.err != io.EOF && !r.closed {
|
if r.err != nil && r.err != io.EOF && !r.isClosed() {
|
||||||
log.Println("rpc: client protocol error:", r.err)
|
log.Println("rpc: client protocol error:", r.err)
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -105,8 +117,8 @@ func (r *rpcStream) Error() error {
|
|||||||
}
|
}
|
||||||
|
|
||||||
func (r *rpcStream) Close() error {
|
func (r *rpcStream) Close() error {
|
||||||
r.Lock()
|
r.once.Do(func() {
|
||||||
defer r.Unlock()
|
close(r.closed)
|
||||||
r.closed = true
|
})
|
||||||
return r.codec.Close()
|
return r.codec.Close()
|
||||||
}
|
}
|
||||||
|
18
cmd/cmd.go
18
cmd/cmd.go
@ -16,6 +16,7 @@ import (
|
|||||||
"time"
|
"time"
|
||||||
|
|
||||||
"github.com/codegangsta/cli"
|
"github.com/codegangsta/cli"
|
||||||
|
log "github.com/golang/glog"
|
||||||
"github.com/micro/go-micro/broker"
|
"github.com/micro/go-micro/broker"
|
||||||
"github.com/micro/go-micro/client"
|
"github.com/micro/go-micro/client"
|
||||||
"github.com/micro/go-micro/registry"
|
"github.com/micro/go-micro/registry"
|
||||||
@ -26,6 +27,8 @@ import (
|
|||||||
)
|
)
|
||||||
|
|
||||||
var (
|
var (
|
||||||
|
Actions = []func(*cli.Context){}
|
||||||
|
|
||||||
Flags = []cli.Flag{
|
Flags = []cli.Flag{
|
||||||
cli.StringFlag{
|
cli.StringFlag{
|
||||||
Name: "server_name",
|
Name: "server_name",
|
||||||
@ -100,9 +103,9 @@ var (
|
|||||||
},
|
},
|
||||||
|
|
||||||
cli.BoolFlag{
|
cli.BoolFlag{
|
||||||
Name: "disable_ping",
|
Name: "enable_ping",
|
||||||
EnvVar: "MICRO_DISABLE_PING",
|
EnvVar: "MICRO_ENABLE_PING",
|
||||||
Usage: "Disable ping",
|
Usage: "Enable ping",
|
||||||
},
|
},
|
||||||
|
|
||||||
// logging flags
|
// logging flags
|
||||||
@ -180,6 +183,7 @@ func ping() {
|
|||||||
cl := &http.Client{}
|
cl := &http.Client{}
|
||||||
|
|
||||||
fn := func() {
|
fn := func() {
|
||||||
|
log.Infof("Ping micro-services.co")
|
||||||
p.Timestamp = time.Now().Unix()
|
p.Timestamp = time.Now().Unix()
|
||||||
b, err := json.Marshal(p)
|
b, err := json.Marshal(p)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
@ -255,7 +259,7 @@ func Setup(c *cli.Context) error {
|
|||||||
|
|
||||||
client.DefaultClient = client.NewClient()
|
client.DefaultClient = client.NewClient()
|
||||||
|
|
||||||
if !c.Bool("disable_ping") {
|
if c.Bool("enable_ping") {
|
||||||
go ping()
|
go ping()
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -283,7 +287,11 @@ GLOBAL OPTIONS:
|
|||||||
app := cli.NewApp()
|
app := cli.NewApp()
|
||||||
app.HideVersion = true
|
app.HideVersion = true
|
||||||
app.Usage = "a go micro app"
|
app.Usage = "a go micro app"
|
||||||
app.Action = func(c *cli.Context) {}
|
app.Action = func(c *cli.Context) {
|
||||||
|
for _, action := range Actions {
|
||||||
|
action(c)
|
||||||
|
}
|
||||||
|
}
|
||||||
app.Before = Setup
|
app.Before = Setup
|
||||||
app.Flags = Flags
|
app.Flags = Flags
|
||||||
app.RunAndExitOnError()
|
app.RunAndExitOnError()
|
||||||
|
52
examples/pubsub/consumer/consumer.go
Normal file
52
examples/pubsub/consumer/consumer.go
Normal file
@ -0,0 +1,52 @@
|
|||||||
|
package main
|
||||||
|
|
||||||
|
import (
|
||||||
|
"fmt"
|
||||||
|
|
||||||
|
log "github.com/golang/glog"
|
||||||
|
"github.com/micro/go-micro/broker"
|
||||||
|
"github.com/micro/go-micro/cmd"
|
||||||
|
|
||||||
|
// To enable rabbitmq plugin uncomment
|
||||||
|
//_ "github.com/micro/go-plugins/broker/rabbitmq"
|
||||||
|
)
|
||||||
|
|
||||||
|
var (
|
||||||
|
topic = "go.micro.topic.foo"
|
||||||
|
)
|
||||||
|
|
||||||
|
// Example of a shared subscription which receives a subset of messages
|
||||||
|
func sharedSub() {
|
||||||
|
_, err := broker.Subscribe(topic, func(p broker.Publication) error {
|
||||||
|
fmt.Println("[sub] received message:", string(p.Message().Body), "header", p.Message().Header)
|
||||||
|
return nil
|
||||||
|
}, broker.QueueName("consumer"))
|
||||||
|
if err != nil {
|
||||||
|
fmt.Println(err)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// Example of a subscription which receives all the messages
|
||||||
|
func sub() {
|
||||||
|
_, err := broker.Subscribe(topic, func(p broker.Publication) error {
|
||||||
|
fmt.Println("[sub] received message:", string(p.Message().Body), "header", p.Message().Header)
|
||||||
|
return nil
|
||||||
|
})
|
||||||
|
if err != nil {
|
||||||
|
fmt.Println(err)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
func main() {
|
||||||
|
cmd.Init()
|
||||||
|
|
||||||
|
if err := broker.Init(); err != nil {
|
||||||
|
log.Fatalf("Broker Init error: %v", err)
|
||||||
|
}
|
||||||
|
if err := broker.Connect(); err != nil {
|
||||||
|
log.Fatalf("Broker Connect error: %v", err)
|
||||||
|
}
|
||||||
|
|
||||||
|
sub()
|
||||||
|
select {}
|
||||||
|
}
|
@ -33,8 +33,9 @@ func pub() {
|
|||||||
}
|
}
|
||||||
|
|
||||||
func sub() {
|
func sub() {
|
||||||
_, err := broker.Subscribe(topic, func(msg *broker.Message) {
|
_, err := broker.Subscribe(topic, func(p broker.Publication) error {
|
||||||
fmt.Println("[sub] received message:", string(msg.Body), "header", msg.Header)
|
fmt.Println("[sub] received message:", string(p.Message().Body), "header", p.Message().Header)
|
||||||
|
return nil
|
||||||
})
|
})
|
||||||
if err != nil {
|
if err != nil {
|
||||||
fmt.Println(err)
|
fmt.Println(err)
|
||||||
|
50
examples/pubsub/producer/producer.go
Normal file
50
examples/pubsub/producer/producer.go
Normal file
@ -0,0 +1,50 @@
|
|||||||
|
package main
|
||||||
|
|
||||||
|
import (
|
||||||
|
"fmt"
|
||||||
|
"time"
|
||||||
|
|
||||||
|
log "github.com/golang/glog"
|
||||||
|
"github.com/micro/go-micro/broker"
|
||||||
|
"github.com/micro/go-micro/cmd"
|
||||||
|
|
||||||
|
// To enable rabbitmq plugin uncomment
|
||||||
|
//_ "github.com/micro/go-plugins/broker/rabbitmq"
|
||||||
|
)
|
||||||
|
|
||||||
|
var (
|
||||||
|
topic = "go.micro.topic.foo"
|
||||||
|
)
|
||||||
|
|
||||||
|
func pub() {
|
||||||
|
tick := time.NewTicker(time.Second)
|
||||||
|
i := 0
|
||||||
|
for _ = range tick.C {
|
||||||
|
msg := &broker.Message{
|
||||||
|
Header: map[string]string{
|
||||||
|
"id": fmt.Sprintf("%d", i),
|
||||||
|
},
|
||||||
|
Body: []byte(fmt.Sprintf("%d: %s", i, time.Now().String())),
|
||||||
|
}
|
||||||
|
if err := broker.Publish(topic, msg); err != nil {
|
||||||
|
log.Errorf("[pub] failed: %v", err)
|
||||||
|
} else {
|
||||||
|
fmt.Println("[pub] pubbed message:", string(msg.Body))
|
||||||
|
}
|
||||||
|
i++
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
func main() {
|
||||||
|
cmd.Init()
|
||||||
|
|
||||||
|
if err := broker.Init(); err != nil {
|
||||||
|
log.Fatalf("Broker Init error: %v", err)
|
||||||
|
}
|
||||||
|
|
||||||
|
if err := broker.Connect(); err != nil {
|
||||||
|
log.Fatalf("Broker Connect error: %v", err)
|
||||||
|
}
|
||||||
|
|
||||||
|
pub()
|
||||||
|
}
|
@ -20,7 +20,10 @@ func init() {
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
func extractValue(v reflect.Type) *registry.Value {
|
func extractValue(v reflect.Type, d int) *registry.Value {
|
||||||
|
if d == 3 {
|
||||||
|
return nil
|
||||||
|
}
|
||||||
if v == nil {
|
if v == nil {
|
||||||
return nil
|
return nil
|
||||||
}
|
}
|
||||||
@ -37,7 +40,10 @@ func extractValue(v reflect.Type) *registry.Value {
|
|||||||
switch v.Kind() {
|
switch v.Kind() {
|
||||||
case reflect.Struct:
|
case reflect.Struct:
|
||||||
for i := 0; i < v.NumField(); i++ {
|
for i := 0; i < v.NumField(); i++ {
|
||||||
val := extractValue(v.Field(i).Type)
|
val := extractValue(v.Field(i).Type, d+1)
|
||||||
|
if val == nil {
|
||||||
|
continue
|
||||||
|
}
|
||||||
val.Name = v.Field(i).Name
|
val.Name = v.Field(i).Name
|
||||||
arg.Values = append(arg.Values, val)
|
arg.Values = append(arg.Values, val)
|
||||||
}
|
}
|
||||||
@ -47,7 +53,10 @@ func extractValue(v reflect.Type) *registry.Value {
|
|||||||
p = p.Elem()
|
p = p.Elem()
|
||||||
}
|
}
|
||||||
arg.Type = "[]" + p.Name()
|
arg.Type = "[]" + p.Name()
|
||||||
arg.Values = append(arg.Values, extractValue(v.Elem()))
|
val := extractValue(v.Elem(), d+1)
|
||||||
|
if val != nil {
|
||||||
|
arg.Values = append(arg.Values, val)
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
return arg
|
return arg
|
||||||
@ -77,8 +86,8 @@ func extractEndpoint(method reflect.Method) *registry.Endpoint {
|
|||||||
stream = true
|
stream = true
|
||||||
}
|
}
|
||||||
|
|
||||||
request := extractValue(reqType)
|
request := extractValue(reqType, 0)
|
||||||
response := extractValue(rspType)
|
response := extractValue(rspType, 0)
|
||||||
|
|
||||||
return ®istry.Endpoint{
|
return ®istry.Endpoint{
|
||||||
Name: method.Name,
|
Name: method.Name,
|
||||||
@ -102,7 +111,7 @@ func extractSubValue(typ reflect.Type) *registry.Value {
|
|||||||
default:
|
default:
|
||||||
return nil
|
return nil
|
||||||
}
|
}
|
||||||
return extractValue(reqType)
|
return extractValue(reqType, 0)
|
||||||
}
|
}
|
||||||
|
|
||||||
func extractAddress(addr string) (string, error) {
|
func extractAddress(addr string) (string, error) {
|
||||||
|
@ -25,6 +25,7 @@ type options struct {
|
|||||||
func newOptions(opt ...Option) options {
|
func newOptions(opt ...Option) options {
|
||||||
opts := options{
|
opts := options{
|
||||||
codecs: make(map[string]codec.NewCodec),
|
codecs: make(map[string]codec.NewCodec),
|
||||||
|
metadata: map[string]string{},
|
||||||
}
|
}
|
||||||
|
|
||||||
for _, o := range opt {
|
for _, o := range opt {
|
||||||
|
@ -2,6 +2,7 @@ package server
|
|||||||
|
|
||||||
import (
|
import (
|
||||||
"fmt"
|
"fmt"
|
||||||
|
"runtime/debug"
|
||||||
"strconv"
|
"strconv"
|
||||||
"strings"
|
"strings"
|
||||||
"sync"
|
"sync"
|
||||||
@ -43,6 +44,13 @@ func newRpcServer(opts ...Option) Server {
|
|||||||
}
|
}
|
||||||
|
|
||||||
func (s *rpcServer) accept(sock transport.Socket) {
|
func (s *rpcServer) accept(sock transport.Socket) {
|
||||||
|
defer func() {
|
||||||
|
if r := recover(); r != nil {
|
||||||
|
log.Error(r, string(debug.Stack()))
|
||||||
|
sock.Close()
|
||||||
|
}
|
||||||
|
}()
|
||||||
|
|
||||||
var msg transport.Message
|
var msg transport.Message
|
||||||
if err := sock.Recv(&msg); err != nil {
|
if err := sock.Recv(&msg); err != nil {
|
||||||
return
|
return
|
||||||
|
@ -155,11 +155,12 @@ func validateSubscriber(sub Subscriber) error {
|
|||||||
}
|
}
|
||||||
|
|
||||||
func (s *rpcServer) createSubHandler(sb *subscriber, opts options) broker.Handler {
|
func (s *rpcServer) createSubHandler(sb *subscriber, opts options) broker.Handler {
|
||||||
return func(msg *broker.Message) {
|
return func(p broker.Publication) error {
|
||||||
|
msg := p.Message()
|
||||||
ct := msg.Header["Content-Type"]
|
ct := msg.Header["Content-Type"]
|
||||||
cf, err := s.newCodec(ct)
|
cf, err := s.newCodec(ct)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return
|
return err
|
||||||
}
|
}
|
||||||
|
|
||||||
hdr := make(map[string]string)
|
hdr := make(map[string]string)
|
||||||
@ -190,11 +191,11 @@ func (s *rpcServer) createSubHandler(sb *subscriber, opts options) broker.Handle
|
|||||||
defer co.Close()
|
defer co.Close()
|
||||||
|
|
||||||
if err := co.ReadHeader(&codec.Message{}, codec.Publication); err != nil {
|
if err := co.ReadHeader(&codec.Message{}, codec.Publication); err != nil {
|
||||||
continue
|
return err
|
||||||
}
|
}
|
||||||
|
|
||||||
if err := co.ReadBody(req.Interface()); err != nil {
|
if err := co.ReadBody(req.Interface()); err != nil {
|
||||||
continue
|
return err
|
||||||
}
|
}
|
||||||
|
|
||||||
fn := func(ctx context.Context, msg Publication) error {
|
fn := func(ctx context.Context, msg Publication) error {
|
||||||
@ -225,6 +226,7 @@ func (s *rpcServer) createSubHandler(sb *subscriber, opts options) broker.Handle
|
|||||||
message: req.Interface(),
|
message: req.Interface(),
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
return nil
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
Loading…
Reference in New Issue
Block a user