influxdb/cluster/service.go

264 lines
6.3 KiB
Go
Raw Normal View History

2015-05-28 21:47:47 +00:00
package cluster
2015-05-30 20:00:46 +00:00
import (
"encoding/binary"
"fmt"
"io"
"log"
"net"
"os"
"strings"
"sync"
2015-05-29 19:50:05 +00:00
"github.com/influxdb/influxdb/meta"
2015-05-30 20:00:46 +00:00
"github.com/influxdb/influxdb/tsdb"
)
// MaxMessageSize defines how large a message can be before we reject it
const MaxMessageSize = 1024 * 1024 * 1024 // 1GB
2015-06-05 22:54:12 +00:00
// MuxHeader is the header byte used in the TCP mux.
const MuxHeader = 2
2015-05-30 20:00:46 +00:00
// Service processes data received over raw TCP connections.
2015-05-28 21:47:47 +00:00
type Service struct {
2015-06-05 22:54:12 +00:00
mu sync.RWMutex
2015-05-30 20:00:46 +00:00
wg sync.WaitGroup
closing chan struct{}
2015-06-05 22:54:12 +00:00
Listener net.Listener
MetaStore interface {
ShardOwner(shardID uint64) (string, string, *meta.ShardGroupInfo)
}
TSDBStore interface {
CreateShard(database, policy string, shardID uint64) error
WriteToShard(shardID uint64, points []tsdb.Point) error
2015-05-30 20:00:46 +00:00
}
Logger *log.Logger
2015-05-28 21:47:47 +00:00
}
2015-05-30 20:00:46 +00:00
// NewService returns a new instance of Service.
2015-05-29 19:50:05 +00:00
func NewService(c Config) *Service {
2015-05-30 20:00:46 +00:00
return &Service{
closing: make(chan struct{}),
Logger: log.New(os.Stderr, "[tcp] ", log.LstdFlags),
}
2015-05-28 21:47:47 +00:00
}
2015-05-30 20:00:46 +00:00
// Open opens the network listener and begins serving requests.
func (s *Service) Open() error {
// Begin serving conections.
s.wg.Add(1)
go s.serve()
return nil
}
2015-06-03 15:58:39 +00:00
// SetLogger sets the internal logger to the logger passed in.
func (s *Service) SetLogger(l *log.Logger) {
s.Logger = l
}
2015-05-30 20:00:46 +00:00
// serve accepts connections from the listener and handles them.
func (s *Service) serve() {
defer s.wg.Done()
for {
// Check if the service is shutting down.
select {
case <-s.closing:
return
default:
}
// Accept the next connection.
2015-06-05 22:54:12 +00:00
conn, err := s.Listener.Accept()
2015-06-06 04:49:03 +00:00
if err != nil {
if strings.Contains(err.Error(), "connection closed") {
s.Logger.Printf("cluster service accept error: %s", err)
return
}
2015-06-06 05:06:52 +00:00
s.Logger.Printf("accept error: %s", err)
2015-05-30 20:00:46 +00:00
continue
}
// Delegate connection handling to a separate goroutine.
s.wg.Add(1)
go func() {
defer s.wg.Done()
s.handleConn(conn)
}()
}
}
// Close shuts down the listener and waits for all connections to finish.
func (s *Service) Close() error {
2015-06-05 22:54:12 +00:00
if s.Listener != nil {
s.Listener.Close()
2015-05-30 20:00:46 +00:00
}
// Shut down all handlers.
close(s.closing)
// s.wg.Wait() // FIXME(benbjohnson)
2015-05-30 20:00:46 +00:00
return nil
}
// handleConn services an individual TCP connection.
func (s *Service) handleConn(conn net.Conn) {
// Ensure connection is closed when service is closed.
closing := make(chan struct{})
defer close(closing)
go func() {
select {
case <-closing:
case <-s.closing:
}
conn.Close()
}()
s.Logger.Printf("accept remote write connection from %v\n", conn.RemoteAddr())
defer func() {
s.Logger.Printf("close remote write connection from %v\n", conn.RemoteAddr())
}()
2015-05-30 20:00:46 +00:00
for {
// Read type-length-value.
typ, buf, err := ReadTLV(conn)
2015-06-06 04:49:03 +00:00
if err != nil {
if strings.HasSuffix(err.Error(), "EOF") {
return
}
2015-05-30 20:00:46 +00:00
s.Logger.Printf("unable to read type-length-value %s", err)
return
}
// Delegate message processing by type.
switch typ {
case writeShardRequestMessage:
err := s.processWriteShardRequest(buf)
2015-06-06 04:34:30 +00:00
if err != nil {
s.Logger.Printf("process write shard error: %s", err)
}
2015-05-30 20:00:46 +00:00
s.writeShardResponse(conn, err)
2015-06-06 04:34:30 +00:00
default:
s.Logger.Printf("cluster service message type not found: %d", typ)
2015-05-30 20:00:46 +00:00
}
}
}
func (s *Service) processWriteShardRequest(buf []byte) error {
// Build request
var req WriteShardRequest
if err := req.UnmarshalBinary(buf); err != nil {
return err
}
err := s.TSDBStore.WriteToShard(req.ShardID(), req.Points())
// We may have received a write for a shard that we don't have locally because the
// sending node may have just created the shard (via the metastore) and the write
// arrived before the local store could create the shard. In this case, we need
// to check the metastore to determine what database and retention policy this
// shard should reside within.
if err == tsdb.ErrShardNotFound {
// Query the metastore for the owner of this shard
database, retentionPolicy, sgi := s.MetaStore.ShardOwner(req.ShardID())
if sgi == nil {
// If we can't find it, then we need to drop this request
// as it is no longer valid. This could happen if writes were queued via
// hinted handoff and delivered after a shard group was deleted.
2015-06-06 04:49:03 +00:00
s.Logger.Printf("drop write request: shard=%d", req.ShardID())
return nil
}
err = s.TSDBStore.CreateShard(database, retentionPolicy, req.ShardID())
if err != nil {
return err
}
return s.TSDBStore.WriteToShard(req.ShardID(), req.Points())
}
if err != nil {
return fmt.Errorf("write shard %d: %s", req.ShardID(), err)
2015-05-30 20:00:46 +00:00
}
return nil
}
func (s *Service) writeShardResponse(w io.Writer, e error) {
// Build response.
var resp WriteShardResponse
if e != nil {
resp.SetCode(1)
resp.SetMessage(e.Error())
} else {
resp.SetCode(0)
}
// Marshal response to binary.
buf, err := resp.MarshalBinary()
if err != nil {
s.Logger.Printf("error marshalling shard response: %s", err)
return
}
// Write to connection.
if err := WriteTLV(w, writeShardResponseMessage, buf); err != nil {
s.Logger.Printf("write shard response error: %s", err)
}
}
// ReadTLV reads a type-length-value record from r.
func ReadTLV(r io.Reader) (byte, []byte, error) {
var typ [1]byte
if _, err := io.ReadFull(r, typ[:]); err != nil {
return 0, nil, fmt.Errorf("read message type: %s", err)
}
// Read the size of the message.
var sz int64
if err := binary.Read(r, binary.BigEndian, &sz); err != nil {
return 0, nil, fmt.Errorf("read message size: %s", err)
}
if sz == 0 {
return 0, nil, fmt.Errorf("invalid message size: %d", sz)
}
if sz >= MaxMessageSize {
return 0, nil, fmt.Errorf("max message size of %d exceeded: %d", MaxMessageSize, sz)
}
2015-05-30 20:00:46 +00:00
// Read the value.
buf := make([]byte, sz)
if _, err := io.ReadFull(r, buf); err != nil {
return 0, nil, fmt.Errorf("read message value: %s", err)
}
return typ[0], buf, nil
}
// WriteTLV writes a type-length-value record to w.
func WriteTLV(w io.Writer, typ byte, buf []byte) error {
if _, err := w.Write([]byte{typ}); err != nil {
return fmt.Errorf("write message type: %s", err)
}
// Write the size of the message.
if err := binary.Write(w, binary.BigEndian, int64(len(buf))); err != nil {
return fmt.Errorf("write message size: %s", err)
}
// Write the value.
if _, err := w.Write(buf); err != nil {
return fmt.Errorf("write message value: %s", err)
}
return nil
}