Ensure that setConfig uses latest functionality (#6302)

master
Harshavardhana 6 years ago committed by kannappanr
parent 50a817e3d3
commit 9f14433cbd
  1. 217
      cmd/admin-handlers.go
  2. 98
      cmd/admin-handlers_test.go
  3. 6
      cmd/admin-router.go
  4. 212
      cmd/admin-rpc-client.go
  5. 25
      cmd/admin-rpc-server.go
  6. 337
      cmd/admin-rpc_test.go
  7. 43
      cmd/config-migrate.go
  8. 14
      cmd/config.go
  9. 30
      cmd/local-admin-client.go
  10. 8
      cmd/local-admin-client_test.go
  11. 8
      cmd/update-notifier_test.go
  12. 41
      pkg/madmin/config-commands.go
  13. 10
      pkg/madmin/generic-commands.go

@ -21,11 +21,8 @@ import (
"context"
"encoding/json"
"errors"
"fmt"
"io"
"net/http"
"net/url"
"path"
"sync"
"time"
@ -38,8 +35,6 @@ import (
)
const (
minioConfigTmpFormat = "config-%s.json"
maxConfigJSONSize = 256 * 1024 // 256KiB
)
@ -48,11 +43,10 @@ type mgmtQueryKey string
// Only valid query params for mgmt admin APIs.
const (
mgmtBucket mgmtQueryKey = "bucket"
mgmtPrefix mgmtQueryKey = "prefix"
mgmtLockOlderThan mgmtQueryKey = "older-than"
mgmtClientToken mgmtQueryKey = "clientToken"
mgmtForceStart mgmtQueryKey = "forceStart"
mgmtBucket mgmtQueryKey = "bucket"
mgmtPrefix mgmtQueryKey = "prefix"
mgmtClientToken mgmtQueryKey = "clientToken"
mgmtForceStart mgmtQueryKey = "forceStart"
)
var (
@ -444,6 +438,15 @@ func (a adminAPIHandlers) HealHandler(w http.ResponseWriter, r *http.Request) {
// GetConfigHandler - GET /minio/admin/v1/config
// Get config.json of this minio setup.
func (a adminAPIHandlers) GetConfigHandler(w http.ResponseWriter, r *http.Request) {
ctx := newContext(r, w, "GetConfigHandler")
// Get current object layer instance.
objectAPI := newObjectLayerFn()
if objectAPI == nil {
writeErrorResponseJSON(w, ErrServerNotInitialized, r.URL)
return
}
// Validate request signature.
adminAPIErr := checkAdminRequestAuthType(r, "")
if adminAPIErr != ErrNone {
@ -451,29 +454,23 @@ func (a adminAPIHandlers) GetConfigHandler(w http.ResponseWriter, r *http.Reques
return
}
// Take a read lock on minio/config.json. NB minio is a
// reserved bucket name and wouldn't conflict with normal
// object operations.
configLock := globalNSMutex.NewNSLock(minioReservedBucket, minioConfigFile)
if configLock.GetRLock(globalObjectTimeout) != nil {
writeErrorResponseJSON(w, ErrOperationTimedOut, r.URL)
config, err := readServerConfig(ctx, objectAPI)
if err != nil {
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
defer configLock.RUnlock()
// Get config.json - in distributed mode, the configuration
// occurring on a quorum of the servers is returned.
configData, err := getPeerConfig(globalAdminPeers)
configData, err := json.Marshal(config)
if err != nil {
logger.LogIf(context.Background(), err)
logger.LogIf(ctx, err)
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
password := globalServerConfig.GetCredential().SecretKey
password := config.GetCredential().SecretKey
econfigData, err := madmin.EncryptServerConfigData(password, configData)
if err != nil {
logger.LogIf(context.Background(), err)
logger.LogIf(ctx, err)
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
@ -492,61 +489,10 @@ func toAdminAPIErrCode(err error) APIErrorCode {
}
}
// SetConfigResult - represents detailed results of a set-config
// operation.
type nodeSummary struct {
Name string `json:"name"`
ErrSet bool `json:"errSet"`
ErrMsg string `json:"errMsg"`
}
type setConfigResult struct {
NodeResults []nodeSummary `json:"nodeResults"`
Status bool `json:"status"`
}
// writeSetConfigResponse - writes setConfigResult value as json
// depending on the status.
func writeSetConfigResponse(w http.ResponseWriter, peers adminPeers,
errs []error, status bool, reqURL *url.URL) {
var nodeResults []nodeSummary
// Build nodeResults based on error values received during
// set-config operation.
for i := range errs {
nodeResults = append(nodeResults, nodeSummary{
Name: peers[i].addr,
ErrSet: errs[i] != nil,
ErrMsg: fmt.Sprintf("%v", errs[i]),
})
}
result := setConfigResult{
Status: status,
NodeResults: nodeResults,
}
// The following elaborate json encoding is to avoid escaping
// '<', '>' in <nil>. Note: json.Encoder.Encode() adds a
// gratuitous "\n".
var resultBuf bytes.Buffer
enc := json.NewEncoder(&resultBuf)
enc.SetEscapeHTML(false)
jsonErr := enc.Encode(result)
if jsonErr != nil {
writeErrorResponseJSON(w, toAPIErrorCode(jsonErr), reqURL)
return
}
writeSuccessResponseJSON(w, resultBuf.Bytes())
return
}
// SetConfigHandler - PUT /minio/admin/v1/config
func (a adminAPIHandlers) SetConfigHandler(w http.ResponseWriter, r *http.Request) {
ctx := newContext(r, w, "SetConfigHandler")
ctx := context.Background()
// Get current object layer instance.
objectAPI := newObjectLayerFn()
if objectAPI == nil {
@ -554,12 +500,6 @@ func (a adminAPIHandlers) SetConfigHandler(w http.ResponseWriter, r *http.Reques
return
}
// Deny if WORM is enabled
if globalWORMEnabled {
writeErrorResponseJSON(w, ErrMethodNotAllowed, r.URL)
return
}
// Validate request signature.
adminAPIErr := checkAdminRequestAuthType(r, "")
if adminAPIErr != ErrNone {
@ -616,58 +556,45 @@ func (a adminAPIHandlers) SetConfigHandler(w http.ResponseWriter, r *http.Reques
}
}
if err := config.Validate(); err != nil {
if err = config.Validate(); err != nil {
writeCustomErrorResponseJSON(w, ErrAdminConfigBadJSON, err.Error(), r.URL)
return
}
// Write config received from request onto a temporary file on
// all nodes.
tmpFileName := fmt.Sprintf(minioConfigTmpFormat, mustGetUUID())
errs := writeTmpConfigPeers(globalAdminPeers, tmpFileName, configBytes)
// Check if the operation succeeded in quorum or more nodes.
rErr := reduceWriteQuorumErrs(ctx, errs, nil, len(globalAdminPeers)/2+1)
if rErr != nil {
writeSetConfigResponse(w, globalAdminPeers, errs, false, r.URL)
return
}
// Take a lock on minio/config.json. NB minio is a reserved
// bucket name and wouldn't conflict with normal object
// operations.
configLock := globalNSMutex.NewNSLock(minioReservedBucket, minioConfigFile)
if configLock.GetLock(globalObjectTimeout) != nil {
writeErrorResponseJSON(w, ErrOperationTimedOut, r.URL)
return
}
defer configLock.Unlock()
// Rename the temporary config file to config.json
errs = commitConfigPeers(globalAdminPeers, tmpFileName)
rErr = reduceWriteQuorumErrs(ctx, errs, nil, len(globalAdminPeers)/2+1)
if rErr != nil {
writeSetConfigResponse(w, globalAdminPeers, errs, false, r.URL)
if err = saveServerConfig(objectAPI, &config); err != nil {
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
// serverMux (cmd/server-mux.go) implements graceful shutdown,
// where all listeners are closed and process restart/shutdown
// happens after 5s or completion of all ongoing http
// requests, whichever is earlier.
writeSetConfigResponse(w, globalAdminPeers, errs, true, r.URL)
// Reply to the client before restarting minio server.
writeSuccessResponseHeadersOnly(w)
// Restart all node for the modified config to take effect.
sendServiceCmd(globalAdminPeers, serviceRestart)
}
// ConfigCredsHandler - POST /minio/admin/v1/config/credential
// UpdateCredsHandler - POST /minio/admin/v1/config/credential
// ----------
// Update credentials in a minio server. In a distributed setup,
// update all the servers in the cluster.
func (a adminAPIHandlers) UpdateCredentialsHandler(w http.ResponseWriter,
r *http.Request) {
ctx := newContext(r, w, "UpdateCredentialsHandler")
// Get current object layer instance.
objectAPI := newObjectLayerFn()
if objectAPI == nil {
writeErrorResponseJSON(w, ErrServerNotInitialized, r.URL)
return
}
// Avoid setting new credentials when they are already passed
// by the environment. Deny if WORM is enabled.
if globalIsEnvCreds {
writeErrorResponseJSON(w, ErrMethodNotAllowed, r.URL)
return
}
// Authenticate request
adminAPIErr := checkAdminRequestAuthType(r, "")
if adminAPIErr != ErrNone {
@ -675,17 +602,32 @@ func (a adminAPIHandlers) UpdateCredentialsHandler(w http.ResponseWriter,
return
}
// Avoid setting new credentials when they are already passed
// by the environment. Deny if WORM is enabled.
if globalIsEnvCreds || globalWORMEnabled {
writeErrorResponseJSON(w, ErrMethodNotAllowed, r.URL)
// Read configuration bytes from request body.
configBuf := make([]byte, maxConfigJSONSize+1)
n, err := io.ReadFull(r.Body, configBuf)
if err == nil {
// More than maxConfigSize bytes were available
writeErrorResponseJSON(w, ErrAdminConfigTooLarge, r.URL)
return
}
if err != io.ErrUnexpectedEOF {
logger.LogIf(ctx, err)
writeErrorResponseJSON(w, toAPIErrorCode(err), r.URL)
return
}
password := globalServerConfig.GetCredential().SecretKey
configBytes, err := madmin.DecryptServerConfigData(password, bytes.NewReader(configBuf[:n]))
if err != nil {
logger.LogIf(ctx, err)
writeErrorResponseJSON(w, ErrAdminConfigBadJSON, r.URL)
return
}
// Decode request body
var req madmin.SetCredsReq
if err := json.NewDecoder(r.Body).Decode(&req); err != nil {
logger.LogIf(context.Background(), err)
if err = json.Unmarshal(configBytes, &req); err != nil {
logger.LogIf(ctx, err)
writeErrorResponseJSON(w, ErrRequestBodyParse, r.URL)
return
}
@ -696,15 +638,6 @@ func (a adminAPIHandlers) UpdateCredentialsHandler(w http.ResponseWriter,
return
}
// Take a lock on minio/config.json. Prevents concurrent
// config file updates.
configLock := globalNSMutex.NewNSLock(minioReservedBucket, minioConfigFile)
if configLock.GetLock(globalObjectTimeout) != nil {
writeErrorResponseJSON(w, ErrOperationTimedOut, r.URL)
return
}
defer configLock.Unlock()
// Acquire lock before updating global configuration.
globalServerConfigMu.Lock()
defer globalServerConfigMu.Unlock()
@ -712,32 +645,18 @@ func (a adminAPIHandlers) UpdateCredentialsHandler(w http.ResponseWriter,
// Update local credentials in memory.
globalServerConfig.SetCredential(creds)
// Construct path to config.json for the given bucket.
configFile := path.Join(bucketConfigPrefix, minioConfigFile)
transactionConfigFile := configFile + ".transaction"
// As object layer's GetObject() and PutObject() take respective lock on minioMetaBucket
// and configFile, take a transaction lock to avoid race.
objLock := globalNSMutex.NewNSLock(minioMetaBucket, transactionConfigFile)
if err = objLock.GetLock(globalOperationTimeout); err != nil {
if err = saveServerConfig(objectAPI, globalServerConfig); err != nil {
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
if err = saveServerConfig(newObjectLayerFn(), globalServerConfig); err != nil {
objLock.Unlock()
writeErrorResponseJSON(w, toAdminAPIErrCode(err), r.URL)
return
}
objLock.Unlock()
// Notify all other Minio peers to update credentials
for host, err := range globalNotificationSys.LoadCredentials() {
reqInfo := (&logger.ReqInfo{}).AppendTags("peerAddress", host.String())
ctx := logger.SetReqInfo(context.Background(), reqInfo)
ctx := logger.SetReqInfo(ctx, reqInfo)
logger.LogIf(ctx, err)
}
// At this stage, the operation is successful, return 200 OK
w.WriteHeader(http.StatusOK)
// Reply to the client before restarting minio server.
writeSuccessResponseHeadersOnly(w)
}

@ -26,7 +26,6 @@ import (
"net/http"
"net/http/httptest"
"net/url"
"os"
"strings"
"testing"
"time"
@ -593,8 +592,14 @@ func TestServiceSetCreds(t *testing.T) {
if err != nil {
t.Fatalf("JSONify err: %v", err)
}
ebody, err := madmin.EncryptServerConfigData(credentials.SecretKey, body)
if err != nil {
t.Fatal(err)
}
// Construct setCreds request
req, err := getServiceCmdRequest(setCreds, credentials, body)
req, err := getServiceCmdRequest(setCreds, credentials, ebody)
if err != nil {
t.Fatalf("Failed to build service status request %v", err)
}
@ -712,16 +717,6 @@ func TestSetConfigHandler(t *testing.T) {
t.Errorf("Expected to succeed but failed with %d", rec.Code)
}
result := setConfigResult{}
err = json.NewDecoder(rec.Body).Decode(&result)
if err != nil {
t.Fatalf("Failed to decode set config result json %v", err)
}
if !result.Status {
t.Error("Expected set-config to succeed, but failed")
}
// Check that a very large config file returns an error.
{
// Make a large enough config string
@ -842,85 +837,6 @@ func TestToAdminAPIErr(t *testing.T) {
}
}
func TestWriteSetConfigResponse(t *testing.T) {
objLayer, fsDir, err := prepareFS()
if err != nil {
t.Fatal(err)
}
defer os.RemoveAll(fsDir)
if err = newTestConfig(globalMinioDefaultRegion, objLayer); err != nil {
t.Fatalf("unable initialize config file, %s", err)
}
testCases := []struct {
status bool
errs []error
}{
// 1. all nodes returned success.
{
status: true,
errs: []error{nil, nil, nil, nil},
},
// 2. some nodes returned errors.
{
status: false,
errs: []error{errDiskNotFound, nil, errDiskAccessDenied, errFaultyDisk},
},
}
testPeers := []adminPeer{
{
addr: "localhost:9001",
},
{
addr: "localhost:9002",
},
{
addr: "localhost:9003",
},
{
addr: "localhost:9004",
},
}
testURL, err := url.Parse("http://dummy.com")
if err != nil {
t.Fatalf("Failed to parse a place-holder url")
}
var actualResult setConfigResult
for i, test := range testCases {
rec := httptest.NewRecorder()
writeSetConfigResponse(rec, testPeers, test.errs, test.status, testURL)
resp := rec.Result()
jsonBytes, err := ioutil.ReadAll(resp.Body)
if err != nil {
t.Fatalf("Test %d: Failed to read response %v", i+1, err)
}
err = json.Unmarshal(jsonBytes, &actualResult)
if err != nil {
t.Fatalf("Test %d: Failed to unmarshal json %v", i+1, err)
}
if actualResult.Status != test.status {
t.Errorf("Test %d: Expected status %v but received %v", i+1, test.status, actualResult.Status)
}
for p, res := range actualResult.NodeResults {
if res.Name != testPeers[p].addr {
t.Errorf("Test %d: Expected node name %s but received %s", i+1, testPeers[p].addr, res.Name)
}
expectedErrMsg := fmt.Sprintf("%v", test.errs[p])
if res.ErrMsg != expectedErrMsg {
t.Errorf("Test %d: Expected error %s but received %s", i+1, expectedErrMsg, res.ErrMsg)
}
expectedErrSet := test.errs[p] != nil
if res.ErrSet != expectedErrSet {
t.Errorf("Test %d: Expected ErrSet %v but received %v", i+1, expectedErrSet, res.ErrSet)
}
}
}
}
func mkHealStartReq(t *testing.T, bucket, prefix string,
opts madmin.HealOpts) *http.Request {

@ -63,9 +63,9 @@ func registerAdminRouter(router *mux.Router) {
/// Config operations
// Update credentials
adminV1Router.Methods(http.MethodPut).Path("/config/credential").HandlerFunc(httpTraceAll(adminAPI.UpdateCredentialsHandler))
adminV1Router.Methods(http.MethodPut).Path("/config/credential").HandlerFunc(httpTraceHdrs(adminAPI.UpdateCredentialsHandler))
// Get config
adminV1Router.Methods(http.MethodGet).Path("/config").HandlerFunc(httpTraceAll(adminAPI.GetConfigHandler))
adminV1Router.Methods(http.MethodGet).Path("/config").HandlerFunc(httpTraceHdrs(adminAPI.GetConfigHandler))
// Set config
adminV1Router.Methods(http.MethodPut).Path("/config").HandlerFunc(httpTraceAll(adminAPI.SetConfigHandler))
adminV1Router.Methods(http.MethodPut).Path("/config").HandlerFunc(httpTraceHdrs(adminAPI.SetConfigHandler))
}

@ -19,7 +19,6 @@ package cmd
import (
"context"
"crypto/tls"
"encoding/json"
"fmt"
"net"
"sort"
@ -69,29 +68,6 @@ func (rpcClient *AdminRPCClient) GetConfig() ([]byte, error) {
return reply, err
}
// WriteTmpConfig - writes config file content to a temporary file on a remote node.
func (rpcClient *AdminRPCClient) WriteTmpConfig(tmpFileName string, configBytes []byte) error {
args := WriteConfigArgs{
TmpFileName: tmpFileName,
Buf: configBytes,
}
reply := VoidReply{}
err := rpcClient.Call(adminServiceName+".WriteTmpConfig", &args, &reply)
logger.LogIf(context.Background(), err)
return err
}
// CommitConfig - Move the new config in tmpFileName onto config.json on a remote node.
func (rpcClient *AdminRPCClient) CommitConfig(tmpFileName string) error {
args := CommitConfigArgs{FileName: tmpFileName}
reply := VoidReply{}
err := rpcClient.Call(adminServiceName+".CommitConfig", &args, &reply)
logger.LogIf(context.Background(), err)
return err
}
// NewAdminRPCClient - returns new admin RPC client.
func NewAdminRPCClient(host *xnet.Host) (*AdminRPCClient, error) {
scheme := "http"
@ -136,8 +112,6 @@ type adminCmdRunner interface {
ReInitFormat(dryRun bool) error
ServerInfo() (ServerInfoData, error)
GetConfig() ([]byte, error)
WriteTmpConfig(tmpFileName string, configBytes []byte) error
CommitConfig(tmpFileName string) error
}
// adminPeer - represents an entity that implements admin API RPCs.
@ -301,189 +275,3 @@ func getPeerUptimes(peers adminPeers) (time.Duration, error) {
return latestUptime, nil
}
// getPeerConfig - Fetches config.json from all nodes in the setup and
// returns the one that occurs in a majority of them.
func getPeerConfig(peers adminPeers) ([]byte, error) {
if !globalIsDistXL {
return peers[0].cmdRunner.GetConfig()
}
errs := make([]error, len(peers))
configs := make([][]byte, len(peers))
// Get config from all servers.
wg := sync.WaitGroup{}
for i, peer := range peers {
wg.Add(1)
go func(idx int, peer adminPeer) {
defer wg.Done()
configs[idx], errs[idx] = peer.cmdRunner.GetConfig()
}(i, peer)
}
wg.Wait()
// Find the maximally occurring config among peers in a
// distributed setup.
serverConfigs := make([]serverConfig, len(peers))
for i, configBytes := range configs {
if errs[i] != nil {
continue
}
// Unmarshal the received config files.
err := json.Unmarshal(configBytes, &serverConfigs[i])
if err != nil {
reqInfo := (&logger.ReqInfo{}).AppendTags("peerAddress", peers[i].addr)
ctx := logger.SetReqInfo(context.Background(), reqInfo)
logger.LogIf(ctx, err)
return nil, err
}
}
configJSON, err := getValidServerConfig(serverConfigs, errs)
if err != nil {
logger.LogIf(context.Background(), err)
return nil, err
}
// Return the config.json that was present quorum or more
// number of disks.
return json.Marshal(configJSON)
}
// getValidServerConfig - finds the server config that is present in
// quorum or more number of servers.
func getValidServerConfig(serverConfigs []serverConfig, errs []error) (scv serverConfig, e error) {
// majority-based quorum
quorum := len(serverConfigs)/2 + 1
// Count the number of disks a config.json was found in.
configCounter := make([]int, len(serverConfigs))
// We group equal serverConfigs by the lowest index of the
// same value; e.g, let us take the following serverConfigs
// in a 4-node setup,
// serverConfigs == [c1, c2, c1, c1]
// configCounter == [3, 1, 0, 0]
// c1, c2 are the only distinct values that appear. c1 is
// identified by 0, the lowest index it appears in and c2 is
// identified by 1. So, we need to find the number of times
// each of these distinct values occur.
// Invariants:
// 1. At the beginning of the i-th iteration, the number of
// unique configurations seen so far is equal to the number of
// non-zero counter values in config[:i].
// 2. At the beginning of the i-th iteration, the sum of
// elements of configCounter[:i] is equal to the number of
// non-error configurations seen so far.
// For each of the serverConfig ...
for i := range serverConfigs {
// Skip nodes where getConfig failed.
if errs[i] != nil {
continue
}
// Check if it is equal to any of the configurations
// seen so far. If j == i is reached then we have an
// unseen configuration.
for j := 0; j <= i; j++ {
if j < i && configCounter[j] == 0 {
// serverConfigs[j] is known to be
// equal to a value that was already
// seen. See example above for
// clarity.
continue
}
if j < i && serverConfigs[i].ConfigDiff(&serverConfigs[j]) == "" {
// serverConfigs[i] is equal to
// serverConfigs[j], update
// serverConfigs[j]'s counter since it
// is the lower index.
configCounter[j]++
break
}
if j == i {
// serverConfigs[i] is equal to no
// other value seen before. It is
// unique so far.
configCounter[i] = 1
break
} // else invariants specified above are violated.
}
}
// We find the maximally occurring server config and check if
// there is quorum.
var configJSON serverConfig
maxOccurrence := 0
for i, count := range configCounter {
if maxOccurrence < count {
maxOccurrence = count
configJSON = serverConfigs[i]
}
}
// If quorum nodes don't agree.
if maxOccurrence < quorum {
return scv, errXLWriteQuorum
}
return configJSON, nil
}
// Write config contents into a temporary file on all nodes.
func writeTmpConfigPeers(peers adminPeers, tmpFileName string, configBytes []byte) []error {
// For a single-node minio server setup.
if !globalIsDistXL {
err := peers[0].cmdRunner.WriteTmpConfig(tmpFileName, configBytes)
return []error{err}
}
errs := make([]error, len(peers))
// Write config into temporary file on all nodes.
wg := sync.WaitGroup{}
for i, peer := range peers {
wg.Add(1)
go func(idx int, peer adminPeer) {
defer wg.Done()
errs[idx] = peer.cmdRunner.WriteTmpConfig(tmpFileName, configBytes)
}(i, peer)
}
wg.Wait()
// Return bytes written and errors (if any) during writing
// temporary config file.
return errs
}
// Move config contents from the given temporary file onto config.json
// on all nodes.
func commitConfigPeers(peers adminPeers, tmpFileName string) []error {
// For a single-node minio server setup.
if !globalIsDistXL {
return []error{peers[0].cmdRunner.CommitConfig(tmpFileName)}
}
errs := make([]error, len(peers))
// Rename temporary config file into configDir/config.json on
// all nodes.
wg := sync.WaitGroup{}
for i, peer := range peers {
wg.Add(1)
go func(idx int, peer adminPeer) {
defer wg.Done()
errs[idx] = peer.cmdRunner.CommitConfig(tmpFileName)
}(i, peer)
}
wg.Wait()
// Return errors (if any) received during rename.
return errs
}

@ -69,31 +69,6 @@ func (receiver *adminRPCReceiver) ReInitFormat(args *ReInitFormatArgs, reply *Vo
return receiver.local.ReInitFormat(args.DryRun)
}
// WriteConfigArgs - wraps the bytes to be written and temporary file name.
type WriteConfigArgs struct {
AuthArgs
TmpFileName string
Buf []byte
}
// WriteTmpConfig - writes the supplied config contents onto the
// supplied temporary file.
func (receiver *adminRPCReceiver) WriteTmpConfig(args *WriteConfigArgs, reply *VoidReply) error {
return receiver.local.WriteTmpConfig(args.TmpFileName, args.Buf)
}
// CommitConfigArgs - wraps the config file name that needs to be
// committed into config.json on this node.
type CommitConfigArgs struct {
AuthArgs
FileName string
}
// CommitConfig - Renames the temporary file into config.json on this node.
func (receiver *adminRPCReceiver) CommitConfig(args *CommitConfigArgs, reply *VoidReply) error {
return receiver.local.CommitConfig(args.FileName)
}
// NewAdminRPCServer - returns new admin RPC server.
func NewAdminRPCServer() (*xrpc.Server, error) {
rpcServer := xrpc.NewServer()

@ -17,13 +17,8 @@
package cmd
import (
"encoding/json"
"io/ioutil"
"net/http"
"net/http/httptest"
"os"
"path/filepath"
"reflect"
"testing"
"time"
@ -172,81 +167,6 @@ func testAdminCmdRunnerGetConfig(t *testing.T, client adminCmdRunner) {
}
}
func testAdminCmdRunnerWriteTmpConfig(t *testing.T, client adminCmdRunner) {
tmpConfigDir := configDir
defer func() {
configDir = tmpConfigDir
}()
tempDir, err := ioutil.TempDir("", ".AdminCmdRunnerWriteTmpConfig.")
if err != nil {
t.Fatalf("unexpected error %v", err)
}
defer os.RemoveAll(tempDir)
configDir = &ConfigDir{dir: tempDir}
testCases := []struct {
tmpFilename string
configBytes []byte
expectErr bool
}{
{"config1.json", []byte(`{"version":"23","region":"us-west-1a"}`), false},
// Overwrite test.
{"config1.json", []byte(`{"version":"23","region":"us-west-1a","browser":"on"}`), false},
{"config2.json", []byte{}, false},
{"config3.json", nil, false},
}
for i, testCase := range testCases {
err := client.WriteTmpConfig(testCase.tmpFilename, testCase.configBytes)
expectErr := (err != nil)
if expectErr != testCase.expectErr {
t.Fatalf("case %v: expected: %v, got: %v", i+1, testCase.expectErr, expectErr)
}
}
}
func testAdminCmdRunnerCommitConfig(t *testing.T, client adminCmdRunner) {
tmpConfigDir := configDir
defer func() {
configDir = tmpConfigDir
}()
tempDir, err := ioutil.TempDir("", ".AdminCmdRunnerCommitConfig.")
if err != nil {
t.Fatalf("unexpected error %v", err)
}
defer os.RemoveAll(tempDir)
configDir = &ConfigDir{dir: tempDir}
err = ioutil.WriteFile(filepath.Join(tempDir, "config.json"), []byte{}, os.ModePerm)
if err != nil {
t.Fatalf("unexpected error %v", err)
}
err = client.WriteTmpConfig("config1.json", []byte(`{"version":"23","region":"us-west-1a"}`))
if err != nil {
t.Fatalf("unexpected error %v", err)
}
testCases := []struct {
tmpFilename string
expectErr bool
}{
{"config1.json", false},
{"config2.json", true},
}
for i, testCase := range testCases {
err := client.CommitConfig(testCase.tmpFilename)
expectErr := (err != nil)
if expectErr != testCase.expectErr {
t.Fatalf("case %v: expected: %v, got: %v", i+1, testCase.expectErr, expectErr)
}
}
}
func newAdminRPCHTTPServerClient(t *testing.T) (*httptest.Server, *AdminRPCClient, *serverConfig) {
rpcServer, err := NewAdminRPCServer()
if err != nil {
@ -317,260 +237,3 @@ func TestAdminRPCClientGetConfig(t *testing.T) {
testAdminCmdRunnerGetConfig(t, rpcClient)
}
func TestAdminRPCClientWriteTmpConfig(t *testing.T) {
httpServer, rpcClient, prevGlobalServerConfig := newAdminRPCHTTPServerClient(t)
defer httpServer.Close()
defer func() {
globalServerConfig = prevGlobalServerConfig
}()
testAdminCmdRunnerWriteTmpConfig(t, rpcClient)
}
func TestAdminRPCClientCommitConfig(t *testing.T) {
httpServer, rpcClient, prevGlobalServerConfig := newAdminRPCHTTPServerClient(t)
defer httpServer.Close()
defer func() {
globalServerConfig = prevGlobalServerConfig
}()
testAdminCmdRunnerCommitConfig(t, rpcClient)
}
var (
config1 = []byte(`{
"version": "13",
"credential": {
"accessKey": "minio",
"secretKey": "minio123"
},
"region": "us-east-1",
"logger": {
"console": {
"enable": true,
"level": "debug"
},
"file": {
"enable": false,
"fileName": "",
"level": ""
}
},
"notify": {
"amqp": {
"1": {
"enable": false,
"url": "",
"exchange": "",
"routingKey": "",
"exchangeType": "",
"mandatory": false,
"immediate": false,
"durable": false,
"internal": false,
"noWait": false,
"autoDeleted": false
}
},
"nats": {
"1": {
"enable": false,
"address": "",
"subject": "",
"username": "",
"password": "",
"token": "",
"secure": false,
"pingInterval": 0,
"streaming": {
"enable": false,
"clusterID": "",
"clientID": "",
"async": false,
"maxPubAcksInflight": 0
}
}
},
"elasticsearch": {
"1": {
"enable": false,
"url": "",
"index": ""
}
},
"redis": {
"1": {
"enable": false,
"address": "",
"password": "",
"key": ""
}
},
"postgresql": {
"1": {
"enable": false,
"connectionString": "",
"table": "",
"host": "",
"port": "",
"user": "",
"password": "",
"database": ""
}
},
"kafka": {
"1": {
"enable": false,
"brokers": null,
"topic": ""
}
},
"webhook": {
"1": {
"enable": false,
"endpoint": ""
}
}
}
}
`)
// diff from config1 - amqp.Enable is True
config2 = []byte(`{
"version": "13",
"credential": {
"accessKey": "minio",
"secretKey": "minio123"
},
"region": "us-east-1",
"logger": {
"console": {
"enable": true,
"level": "debug"
},
"file": {
"enable": false,
"fileName": "",
"level": ""
}
},
"notify": {
"amqp": {
"1": {
"enable": true,
"url": "",
"exchange": "",
"routingKey": "",
"exchangeType": "",
"mandatory": false,
"immediate": false,
"durable": false,
"internal": false,
"noWait": false,
"autoDeleted": false
}
},
"nats": {
"1": {
"enable": false,
"address": "",
"subject": "",
"username": "",
"password": "",
"token": "",
"secure": false,
"pingInterval": 0,
"streaming": {
"enable": false,
"clusterID": "",
"clientID": "",
"async": false,
"maxPubAcksInflight": 0
}
}
},
"elasticsearch": {
"1": {
"enable": false,
"url": "",
"index": ""
}
},
"redis": {
"1": {
"enable": false,
"address": "",
"password": "",
"key": ""
}
},
"postgresql": {
"1": {
"enable": false,
"connectionString": "",
"table": "",
"host": "",
"port": "",
"user": "",
"password": "",
"database": ""
}
},
"kafka": {
"1": {
"enable": false,
"brokers": null,
"topic": ""
}
},
"webhook": {
"1": {
"enable": false,
"endpoint": ""
}
}
}
}
`)
)
// TestGetValidServerConfig - test for getValidServerConfig.
func TestGetValidServerConfig(t *testing.T) {
var c1, c2 serverConfig
err := json.Unmarshal(config1, &c1)
if err != nil {
t.Fatalf("json unmarshal of %s failed: %v", string(config1), err)
}
err = json.Unmarshal(config2, &c2)
if err != nil {
t.Fatalf("json unmarshal of %s failed: %v", string(config2), err)
}
// Valid config.
noErrs := []error{nil, nil, nil, nil}
serverConfigs := []serverConfig{c1, c2, c1, c1}
validConfig, err := getValidServerConfig(serverConfigs, noErrs)
if err != nil {
t.Errorf("Expected a valid config but received %v instead", err)
}
if !reflect.DeepEqual(validConfig, c1) {
t.Errorf("Expected valid config to be %v but received %v", config1, validConfig)
}
// Invalid config - no quorum.
serverConfigs = []serverConfig{c1, c2, c2, c1}
_, err = getValidServerConfig(serverConfigs, noErrs)
if err != errXLWriteQuorum {
t.Errorf("Expected to fail due to lack of quorum but received %v", err)
}
// All errors
allErrs := []error{errDiskNotFound, errDiskNotFound, errDiskNotFound, errDiskNotFound}
serverConfigs = []serverConfig{{}, {}, {}, {}}
_, err = getValidServerConfig(serverConfigs, allErrs)
if err != errXLWriteQuorum {
t.Errorf("Expected to fail due to lack of quorum but received %v", err)
}
}

@ -17,8 +17,10 @@
package cmd
import (
"context"
"fmt"
"os"
"path"
"path/filepath"
"github.com/minio/minio/cmd/crypto"
@ -49,7 +51,7 @@ func GetVersion(configFile string) (string, error) {
return quick.GetVersion(configFile, globalEtcdClient)
}
// Migrates all config versions from "1" to "18".
// Migrates all config versions from "1" to "28".
func migrateConfig() error {
// Purge all configs with version '1',
// this is a special case since version '1' used
@ -2406,3 +2408,42 @@ func migrateV27ToV28() error {
logger.Info(configMigrateMSGTemplate, configFile, "27", "28")
return nil
}
// Migrates '.minio.sys/config.json' v27 to v28.
func migrateMinioSysConfig(objAPI ObjectLayer) error {
// Construct path to config.json for the given bucket.
configFile := path.Join(bucketConfigPrefix, minioConfigFile)
transactionConfigFile := configFile + ".transaction"
// As object layer's GetObject() and PutObject() take respective lock on minioMetaBucket
// and configFile, take a transaction lock to avoid race.
objLock := globalNSMutex.NewNSLock(minioMetaBucket, transactionConfigFile)
if err := objLock.GetLock(globalOperationTimeout); err != nil {
return err
}
defer objLock.Unlock()
return migrateV27ToV28MinioSys(objAPI)
}
func migrateV27ToV28MinioSys(objAPI ObjectLayer) error {
configFile := path.Join(minioConfigPrefix, minioConfigFile)
srvConfig, err := readServerConfig(context.Background(), objAPI)
if err == errConfigNotFound {
return nil
} else if err != nil {
return fmt.Errorf("Unable to load config file. %v", err)
}
if srvConfig.Version != "27" {
return nil
}
srvConfig.Version = "28"
srvConfig.KMS = crypto.KMSConfig{}
if err = saveServerConfig(objAPI, srvConfig); err != nil {
return fmt.Errorf("Failed to migrate config from ‘27’ to ‘28’. %v", err)
}
logger.Info(configMigrateMSGTemplate, configFile, "27", "28")
return nil
}

@ -246,16 +246,22 @@ func initConfig() {
logger.Fatal(err, "Unable to migrate 'config.json' to '.minio.sys/config/config.json'")
}
}
}
if err := checkServerConfig(context.Background(), newObjectLayerFn()); err != nil {
objAPI := newObjectLayerFn()
if objAPI == nil {
logger.FatalIf(errServerNotInitialized, "Server is not initialized yet unable to proceed")
}
if err := checkServerConfig(context.Background(), objAPI); err != nil {
if err == errConfigNotFound {
// Config file does not exist, we create it fresh and return upon success.
logger.FatalIf(newConfig(newObjectLayerFn()), "Unable to initialize minio config for the first time")
logger.FatalIf(newConfig(objAPI), "Unable to initialize minio config for the first time")
logger.Info("Created minio configuration file successfully at " + getConfigDir())
} else {
logger.FatalIf(err, "Unable to load the configuration file")
}
}
logger.FatalIf(loadConfig(newObjectLayerFn()), "Unable to load the configuration file")
logger.FatalIf(migrateMinioSysConfig(objAPI), "Config migration failed for minio.sys config")
logger.FatalIf(loadConfig(objAPI), "Unable to load the configuration file")
}

@ -20,11 +20,6 @@ import (
"context"
"encoding/json"
"fmt"
"io/ioutil"
"os"
"path/filepath"
"github.com/minio/minio/cmd/logger"
)
// localAdminClient - represents admin operation to be executed locally.
@ -85,28 +80,3 @@ func (lc localAdminClient) GetConfig() ([]byte, error) {
return json.Marshal(globalServerConfig)
}
// WriteTmpConfig - writes config file content to a temporary file on
// the local server.
func (lc localAdminClient) WriteTmpConfig(tmpFileName string, configBytes []byte) error {
tmpConfigFile := filepath.Join(getConfigDir(), tmpFileName)
err := ioutil.WriteFile(tmpConfigFile, configBytes, 0666)
reqInfo := (&logger.ReqInfo{}).AppendTags("tmpConfigFile", tmpConfigFile)
ctx := logger.SetReqInfo(context.Background(), reqInfo)
logger.LogIf(ctx, err)
return err
}
// CommitConfig - Move the new config in tmpFileName onto config.json
// on a local node.
func (lc localAdminClient) CommitConfig(tmpFileName string) error {
configFile := getConfigFile()
tmpConfigFile := filepath.Join(getConfigDir(), tmpFileName)
err := os.Rename(tmpConfigFile, configFile)
reqInfo := (&logger.ReqInfo{}).AppendTags("tmpConfigFile", tmpConfigFile)
reqInfo.AppendTags("configFile", configFile)
ctx := logger.SetReqInfo(context.Background(), reqInfo)
logger.LogIf(ctx, err)
return err
}

@ -35,11 +35,3 @@ func TestLocalAdminClientServerInfo(t *testing.T) {
func TestLocalAdminClientGetConfig(t *testing.T) {
testAdminCmdRunnerGetConfig(t, &localAdminClient{})
}
func TestLocalAdminClientWriteTmpConfig(t *testing.T) {
testAdminCmdRunnerWriteTmpConfig(t, &localAdminClient{})
}
func TestLocalAdminClientCommitConfig(t *testing.T) {
testAdminCmdRunnerCommitConfig(t, &localAdminClient{})
}

@ -21,8 +21,6 @@ import (
"strings"
"testing"
"time"
"github.com/fatih/color"
)
// Tests update notifier string builder.
@ -67,13 +65,11 @@ func TestPrepareUpdateMessage(t *testing.T) {
}
plainMsg := "You are running an older version of Minio released"
yellow := color.New(color.FgYellow, color.Bold).SprintfFunc()
cyan := color.New(color.FgCyan, color.Bold).SprintFunc()
for i, testCase := range testCases {
output := prepareUpdateMessage(testCase.dlURL, testCase.older)
line1 := fmt.Sprintf("%s %s", plainMsg, yellow(testCase.expectedSubStr))
line2 := fmt.Sprintf("Update: %s", cyan(testCase.dlURL))
line1 := fmt.Sprintf("%s %s", plainMsg, colorYellowBold(testCase.expectedSubStr))
line2 := fmt.Sprintf("Update: %s", colorCyanBold(testCase.dlURL))
// Uncomment below to see message appearance:
// fmt.Println(output)
switch {

@ -32,21 +32,6 @@ import (
"golang.org/x/crypto/argon2"
)
// NodeSummary - represents the result of an operation part of
// set-config on a node.
type NodeSummary struct {
Name string `json:"name"`
ErrSet bool `json:"errSet"`
ErrMsg string `json:"errMsg"`
}
// SetConfigResult - represents detailed results of a set-config
// operation.
type SetConfigResult struct {
NodeResults []NodeSummary `json:"nodeResults"`
Status bool `json:"status"`
}
// EncryptServerConfigData - encrypts server config data.
func EncryptServerConfigData(password string, data []byte) ([]byte, error) {
salt := make([]byte, 32)
@ -106,17 +91,17 @@ func (adm *AdminClient) GetConfig() ([]byte, error) {
}
// SetConfig - set config supplied as config.json for the setup.
func (adm *AdminClient) SetConfig(config io.Reader) (r SetConfigResult, err error) {
func (adm *AdminClient) SetConfig(config io.Reader) (err error) {
const maxConfigJSONSize = 256 * 1024 // 256KiB
// Read configuration bytes
configBuf := make([]byte, maxConfigJSONSize+1)
n, err := io.ReadFull(config, configBuf)
if err == nil {
return r, fmt.Errorf("too large file")
return fmt.Errorf("too large file")
}
if err != io.ErrUnexpectedEOF {
return r, err
return err
}
configBytes := configBuf[:n]
@ -127,21 +112,21 @@ func (adm *AdminClient) SetConfig(config io.Reader) (r SetConfigResult, err erro
// Check if read data is in json format
if err = json.Unmarshal(configBytes, &cfg); err != nil {
return r, errors.New("Invalid JSON format: " + err.Error())
return errors.New("Invalid JSON format: " + err.Error())
}
// Check if the provided json file has "version" key set
if cfg.Version == "" {
return r, errors.New("Missing or unset \"version\" key in json file")
return errors.New("Missing or unset \"version\" key in json file")
}
// Validate there are no duplicate keys in the JSON
if err = quick.CheckDuplicateKeys(string(configBytes)); err != nil {
return r, errors.New("Duplicate key in json file: " + err.Error())
return errors.New("Duplicate key in json file: " + err.Error())
}
econfigBytes, err := EncryptServerConfigData(adm.secretAccessKey, configBytes)
if err != nil {
return r, err
return err
}
reqData := requestData{
@ -154,18 +139,12 @@ func (adm *AdminClient) SetConfig(config io.Reader) (r SetConfigResult, err erro
defer closeResponse(resp)
if err != nil {
return r, err
return err
}
if resp.StatusCode != http.StatusOK {
return r, httpRespToErrorResponse(resp)
}
jsonBytes, err := ioutil.ReadAll(resp.Body)
if err != nil {
return r, err
return httpRespToErrorResponse(resp)
}
err = json.Unmarshal(jsonBytes, &r)
return r, err
return nil
}

@ -19,7 +19,6 @@ package madmin
import (
"encoding/json"
"fmt"
"net/http"
)
@ -38,15 +37,15 @@ func (adm *AdminClient) SetCredentials(access, secret string) error {
return err
}
// No TLS?
if !adm.secure {
return fmt.Errorf("credentials cannot be updated over an insecure connection")
ebody, err := EncryptServerConfigData(adm.secretAccessKey, body)
if err != nil {
return err
}
// Setup new request
reqData := requestData{
relPath: "/v1/config/credential",
content: body,
content: ebody,
}
// Execute GET on bucket to list objects.
@ -62,5 +61,6 @@ func (adm *AdminClient) SetCredentials(access, secret string) error {
if resp.StatusCode != http.StatusOK {
return httpRespToErrorResponse(resp)
}
return nil
}

Loading…
Cancel
Save