mirror of
https://github.com/minio/minio.git
synced 2025-11-09 05:34:56 -05:00
Layered rpc-client implementation (#2512)
This commit is contained in:
committed by
Harshavardhana
parent
7e3e24b394
commit
bda6bcd5be
73
vendor/github.com/minio/dsync/drwmutex.go
generated
vendored
73
vendor/github.com/minio/dsync/drwmutex.go
generated
vendored
@@ -19,7 +19,6 @@ package dsync
|
||||
import (
|
||||
"math"
|
||||
"math/rand"
|
||||
"net/rpc"
|
||||
"sync"
|
||||
"time"
|
||||
)
|
||||
@@ -40,6 +39,20 @@ type Granted struct {
|
||||
uid string
|
||||
}
|
||||
|
||||
type LockArgs struct {
|
||||
Token string
|
||||
Timestamp time.Time
|
||||
Name string
|
||||
}
|
||||
|
||||
func (l *LockArgs) SetToken(token string) {
|
||||
l.Token = token
|
||||
}
|
||||
|
||||
func (l *LockArgs) SetTimestamp(tstamp time.Time) {
|
||||
l.Timestamp = tstamp
|
||||
}
|
||||
|
||||
func NewDRWMutex(name string) *DRWMutex {
|
||||
return &DRWMutex{
|
||||
Name: name,
|
||||
@@ -48,28 +61,6 @@ func NewDRWMutex(name string) *DRWMutex {
|
||||
}
|
||||
}
|
||||
|
||||
// Connect to respective lock server nodes on the first Lock() call.
|
||||
func connectLazy() {
|
||||
if clnts == nil {
|
||||
panic("rpc client connections weren't initialized.")
|
||||
}
|
||||
for i := range clnts {
|
||||
if clnts[i].rpc != nil {
|
||||
continue
|
||||
}
|
||||
|
||||
// Pass in unique path (as required by server.HandleHTTP().
|
||||
// Ignore failure to connect, the lock server node may join the
|
||||
// cluster later.
|
||||
clnt, err := rpc.DialHTTPPath("tcp", nodes[i], rpcPaths[i])
|
||||
if err != nil {
|
||||
clnts[i].SetRPC(nil)
|
||||
continue
|
||||
}
|
||||
clnts[i].SetRPC(clnt)
|
||||
}
|
||||
}
|
||||
|
||||
// RLock holds a read lock on dm.
|
||||
//
|
||||
// If the lock is already in use, the calling goroutine
|
||||
@@ -83,7 +74,6 @@ func (dm *DRWMutex) RLock() {
|
||||
runs, backOff := 1, 1
|
||||
|
||||
for {
|
||||
connectLazy()
|
||||
|
||||
// create temp arrays on stack
|
||||
locks := make([]bool, dnodeCount)
|
||||
@@ -128,8 +118,6 @@ func (dm *DRWMutex) Lock() {
|
||||
runs, backOff := 1, 1
|
||||
|
||||
for {
|
||||
connectLazy()
|
||||
|
||||
// create temp arrays on stack
|
||||
locks := make([]bool, dnodeCount)
|
||||
ids := make([]string, dnodeCount)
|
||||
@@ -161,7 +149,7 @@ func (dm *DRWMutex) Lock() {
|
||||
|
||||
// lock tries to acquire the distributed lock, returning true or false
|
||||
//
|
||||
func lock(clnts []*RPCClient, locks *[]bool, uids *[]string, lockName string, isReadLock bool) bool {
|
||||
func lock(clnts []RPC, locks *[]bool, uids *[]string, lockName string, isReadLock bool) bool {
|
||||
|
||||
// Create buffered channel of quorum size
|
||||
ch := make(chan Granted, dquorum)
|
||||
@@ -169,15 +157,15 @@ func lock(clnts []*RPCClient, locks *[]bool, uids *[]string, lockName string, is
|
||||
for index, c := range clnts {
|
||||
|
||||
// broadcast lock request to all nodes
|
||||
go func(index int, isReadLock bool, c *RPCClient) {
|
||||
go func(index int, isReadLock bool, c RPC) {
|
||||
// All client methods issuing RPCs are thread-safe and goroutine-safe,
|
||||
// i.e. it is safe to call them from multiple concurrently running go routines.
|
||||
var status bool
|
||||
var err error
|
||||
if isReadLock {
|
||||
err = c.Call("Dsync.RLock", lockName, &status)
|
||||
err = c.Call("Dsync.RLock", &LockArgs{Name: lockName}, &status)
|
||||
} else {
|
||||
err = c.Call("Dsync.Lock", lockName, &status)
|
||||
err = c.Call("Dsync.Lock", &LockArgs{Name: lockName}, &status)
|
||||
}
|
||||
|
||||
locked, uid := false, ""
|
||||
@@ -185,14 +173,8 @@ func lock(clnts []*RPCClient, locks *[]bool, uids *[]string, lockName string, is
|
||||
locked = status
|
||||
// TODO: Get UIOD again
|
||||
uid = ""
|
||||
} else {
|
||||
// If rpc call failed due to connection related errors, reset rpc.Client object
|
||||
// to trigger reconnect on subsequent Lock()/Unlock() requests to the same node.
|
||||
if IsRPCError(err) {
|
||||
clnts[index].SetRPC(nil)
|
||||
}
|
||||
// silently ignore error, retry later
|
||||
}
|
||||
// silently ignore error, retry later
|
||||
|
||||
ch <- Granted{index: index, locked: locked, uid: uid}
|
||||
|
||||
@@ -277,7 +259,7 @@ func quorumMet(locks *[]bool) bool {
|
||||
}
|
||||
|
||||
// releaseAll releases all locks that are marked as locked
|
||||
func releaseAll(clnts []*RPCClient, locks *[]bool, ids *[]string, lockName string, isReadLock bool) {
|
||||
func releaseAll(clnts []RPC, locks *[]bool, ids *[]string, lockName string, isReadLock bool) {
|
||||
|
||||
for lock := 0; lock < dnodeCount; lock++ {
|
||||
if (*locks)[lock] {
|
||||
@@ -337,38 +319,31 @@ func (dm *DRWMutex) Unlock() {
|
||||
}
|
||||
|
||||
// sendRelease sends a release message to a node that previously granted a lock
|
||||
func sendRelease(c *RPCClient, name, uid string, isReadLock bool) {
|
||||
func sendRelease(c RPC, name, uid string, isReadLock bool) {
|
||||
|
||||
backOffArray := []time.Duration{30 * time.Second, 1 * time.Minute, 3 * time.Minute, 10 * time.Minute, 30 * time.Minute, 1 * time.Hour}
|
||||
|
||||
go func(c *RPCClient, name, uid string) {
|
||||
go func(c RPC, name, uid string) {
|
||||
|
||||
for _, backOff := range backOffArray {
|
||||
|
||||
// Make sure we are connected
|
||||
connectLazy()
|
||||
|
||||
// All client methods issuing RPCs are thread-safe and goroutine-safe,
|
||||
// i.e. it is safe to call them from multiple concurrently running goroutines.
|
||||
var status bool
|
||||
var err error
|
||||
// TODO: Send UID to server
|
||||
if isReadLock {
|
||||
if err = c.Call("Dsync.RUnlock", name, &status); err == nil {
|
||||
if err = c.Call("Dsync.RUnlock", &LockArgs{Name: name}, &status); err == nil {
|
||||
// RUnlock delivered, exit out
|
||||
return
|
||||
}
|
||||
} else {
|
||||
if err = c.Call("Dsync.Unlock", name, &status); err == nil {
|
||||
if err = c.Call("Dsync.Unlock", &LockArgs{Name: name}, &status); err == nil {
|
||||
// Unlock delivered, exit out
|
||||
return
|
||||
}
|
||||
}
|
||||
|
||||
// If rpc call failed due to connection related errors, reset rpc.Client object
|
||||
// to trigger reconnect on subsequent Lock()/Unlock() requests to the same node.
|
||||
c.SetRPC(nil)
|
||||
|
||||
// wait
|
||||
time.Sleep(backOff)
|
||||
}
|
||||
|
||||
29
vendor/github.com/minio/dsync/dsync.go
generated
vendored
29
vendor/github.com/minio/dsync/dsync.go
generated
vendored
@@ -26,43 +26,30 @@ const DefaultPath = "/rpc/dsync"
|
||||
// Number of nodes participating in the distributed locking.
|
||||
var dnodeCount int
|
||||
|
||||
// List of nodes participating.
|
||||
var nodes []string
|
||||
|
||||
// List of rpc paths, one per lock server.
|
||||
var rpcPaths []string
|
||||
|
||||
// List of rpc client objects, one per lock server.
|
||||
var clnts []*RPCClient
|
||||
var clnts []RPC
|
||||
|
||||
// Simple majority based quorum, set to dNodeCount/2+1
|
||||
var dquorum int
|
||||
|
||||
// SetNodesWithPath - initializes package-level global state variables such as
|
||||
// nodes, rpcPaths, clnts.
|
||||
// SetNodesWithPath - initializes package-level global state variables such as clnts.
|
||||
// N B - This function should be called only once inside any program that uses
|
||||
// dsync.
|
||||
func SetNodesWithPath(nodeList []string, paths []string) (err error) {
|
||||
func SetNodesWithClients(rpcClnts []RPC) (err error) {
|
||||
|
||||
// Validate if number of nodes is within allowable range.
|
||||
if dnodeCount != 0 {
|
||||
return errors.New("Cannot reinitialize dsync package")
|
||||
} else if len(nodeList) < 4 {
|
||||
} else if len(rpcClnts) < 4 {
|
||||
return errors.New("Dsync not designed for less than 4 nodes")
|
||||
} else if len(nodeList) > 16 {
|
||||
} else if len(rpcClnts) > 16 {
|
||||
return errors.New("Dsync not designed for more than 16 nodes")
|
||||
}
|
||||
|
||||
nodes = make([]string, len(nodeList))
|
||||
copy(nodes, nodeList[:])
|
||||
rpcPaths = make([]string, len(paths))
|
||||
copy(rpcPaths, paths[:])
|
||||
dnodeCount = len(nodes)
|
||||
dnodeCount = len(rpcClnts)
|
||||
dquorum = dnodeCount/2 + 1
|
||||
clnts = make([]*RPCClient, dnodeCount)
|
||||
// Initialize node name and rpc path for each RPCClient object.
|
||||
for i := range clnts {
|
||||
clnts[i] = newClient(nodes[i], rpcPaths[i])
|
||||
}
|
||||
clnts = make([]RPC, dnodeCount)
|
||||
copy(clnts, rpcClnts)
|
||||
return nil
|
||||
}
|
||||
|
||||
29
vendor/github.com/minio/dsync/rpc-client-interface.go
generated
vendored
Normal file
29
vendor/github.com/minio/dsync/rpc-client-interface.go
generated
vendored
Normal file
@@ -0,0 +1,29 @@
|
||||
/*
|
||||
* Minio Cloud Storage, (C) 2016 Minio, Inc.
|
||||
*
|
||||
* Licensed under the Apache License, Version 2.0 (the "License");
|
||||
* you may not use this file except in compliance with the License.
|
||||
* You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package dsync
|
||||
|
||||
import "time"
|
||||
|
||||
type TokenSetter interface {
|
||||
SetToken(token string)
|
||||
SetTimestamp(tstamp time.Time)
|
||||
}
|
||||
|
||||
type RPC interface {
|
||||
Call(serviceMethod string, args TokenSetter, reply interface{}) error
|
||||
Close() error
|
||||
}
|
||||
77
vendor/github.com/minio/dsync/rpc-client.go
generated
vendored
77
vendor/github.com/minio/dsync/rpc-client.go
generated
vendored
@@ -1,77 +0,0 @@
|
||||
/*
|
||||
* Minio Cloud Storage, (C) 2016 Minio, Inc.
|
||||
*
|
||||
* Licensed under the Apache License, Version 2.0 (the "License");
|
||||
* you may not use this file except in compliance with the License.
|
||||
* You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package dsync
|
||||
|
||||
import (
|
||||
"net/rpc"
|
||||
"reflect"
|
||||
"sync"
|
||||
)
|
||||
|
||||
// Wrapper type for rpc.Client that provides connection management like
|
||||
// reconnect on first failure.
|
||||
type RPCClient struct {
|
||||
sync.Mutex
|
||||
rpc *rpc.Client
|
||||
node string
|
||||
rpcPath string
|
||||
}
|
||||
|
||||
func newClient(node, rpcPath string) *RPCClient {
|
||||
return &RPCClient{
|
||||
node: node,
|
||||
rpcPath: rpcPath,
|
||||
}
|
||||
}
|
||||
|
||||
func (rpcClient *RPCClient) SetRPC(rpc *rpc.Client) {
|
||||
rpcClient.Lock()
|
||||
defer rpcClient.Unlock()
|
||||
rpcClient.rpc = rpc
|
||||
}
|
||||
|
||||
func (rpcClient *RPCClient) Call(serviceMethod string, args interface{}, reply interface{}) error {
|
||||
rpcClient.Lock()
|
||||
defer rpcClient.Unlock()
|
||||
if rpcClient.rpc == nil {
|
||||
return rpc.ErrShutdown
|
||||
}
|
||||
err := rpcClient.rpc.Call(serviceMethod, args, reply)
|
||||
return err
|
||||
|
||||
}
|
||||
|
||||
func (rpcClient *RPCClient) Reconnect() error {
|
||||
rpcClient.Lock()
|
||||
defer rpcClient.Unlock()
|
||||
clnt, err := rpc.DialHTTPPath("tcp", rpcClient.node, rpcClient.rpcPath)
|
||||
if err != nil {
|
||||
return err
|
||||
}
|
||||
rpcClient.rpc = clnt
|
||||
return nil
|
||||
}
|
||||
|
||||
func IsRPCError(err error) bool {
|
||||
// The following are net/rpc specific errors that indicate that
|
||||
// the connection may have been reset.
|
||||
if err == rpc.ErrShutdown ||
|
||||
reflect.TypeOf(err) == reflect.TypeOf((*rpc.ServerError)(nil)).Elem() {
|
||||
return true
|
||||
}
|
||||
return false
|
||||
}
|
||||
Reference in New Issue
Block a user