Add /v1/internal/service-virtual-ip for manually setting service VIPs (#17294)

This commit is contained in:
Kyle Havlovitz 2023-05-11 17:38:52 -07:00 committed by GitHub
parent 09de8cedca
commit b6d5d5649d
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
11 changed files with 489 additions and 26 deletions

View File

@ -573,3 +573,34 @@ RETRY_ONCE:
s.nodeMetricsLabels())
return out.Services, nil
}
func (s *HTTPHandlers) AssignManualServiceVIPs(resp http.ResponseWriter, req *http.Request) (interface{}, error) {
metrics.IncrCounterWithLabels([]string{"client", "api", "service_virtual_ips"}, 1,
s.nodeMetricsLabels())
var args structs.AssignServiceManualVIPsRequest
if err := s.parseEntMetaNoWildcard(req, &args.EnterpriseMeta); err != nil {
return nil, err
}
if err := s.rewordUnknownEnterpriseFieldError(decodeBody(req.Body, &args)); err != nil {
return nil, HTTPError{StatusCode: http.StatusBadRequest, Reason: fmt.Sprintf("Request decode failed: %v", err)}
}
// Setup the default DC if not provided
if args.Datacenter == "" {
args.Datacenter = s.agent.config.Datacenter
}
s.parseToken(req, &args.Token)
// Forward to the servers
var out structs.AssignServiceManualVIPsResponse
if err := s.agent.RPC(req.Context(), "Internal.AssignManualServiceVIPs", &args, &out); err != nil {
metrics.IncrCounterWithLabels([]string{"client", "rpc", "error", "service_virtual_ips"}, 1,
s.nodeMetricsLabels())
return nil, err
}
metrics.IncrCounterWithLabels([]string{"client", "api", "success", "service_virtual_ips"}, 1,
s.nodeMetricsLabels())
return out, nil
}

View File

@ -2028,3 +2028,67 @@ func TestCatalog_GatewayServices_Ingress(t *testing.T) {
require.Equal(r, expect, gatewayServices)
})
}
func TestCatalogRegister_AssignManualServiceVIPs(t *testing.T) {
if testing.Short() {
t.Skip("too slow for testing.Short")
}
t.Parallel()
a := NewTestAgent(t, "")
defer a.Shutdown()
testrpc.WaitForTestAgent(t, a.RPC, "dc1")
for _, service := range []string{"api", "web"} {
req := structs.ConfigEntryRequest{
Datacenter: "dc1",
Entry: &structs.ServiceResolverConfigEntry{
Kind: structs.ServiceResolver,
Name: service,
},
}
var out bool
require.NoError(t, a.RPC(context.Background(), "ConfigEntry.Apply", &req, &out))
}
assignVIPs := func(req structs.AssignServiceManualVIPsRequest, expect structs.AssignServiceManualVIPsResponse) {
httpReq, _ := http.NewRequest("PUT", "/v1/internal/service-virtual-ip", jsonReader(req))
resp := httptest.NewRecorder()
obj, err := a.srv.AssignManualServiceVIPs(resp, httpReq)
require.NoError(t, err)
result, ok := obj.(structs.AssignServiceManualVIPsResponse)
require.True(t, ok)
require.Equal(t, expect, result)
}
// Assign some manual IPs to the service
assignVIPs(structs.AssignServiceManualVIPsRequest{
Service: "api",
ManualVIPs: []string{"1.1.1.1", "2.2.2.2", "3.3.3.3"},
}, structs.AssignServiceManualVIPsResponse{
Found: true,
})
// Assign some manual IPs to the new service, reassigning one from the existing service.
assignVIPs(structs.AssignServiceManualVIPsRequest{
Service: "web",
ManualVIPs: []string{"2.2.2.2", "4.4.4.4"},
}, structs.AssignServiceManualVIPsResponse{
Found: true,
UnassignedFrom: []structs.PeeredServiceName{
{
ServiceName: structs.ServiceName{Name: "api", EnterpriseMeta: *acl.DefaultEnterpriseMeta()},
},
},
})
// Assign some manual IPs a non-existent service, should be a no-op.
assignVIPs(structs.AssignServiceManualVIPsRequest{
Service: "nope",
ManualVIPs: []string{"1.1.1.1", "2.2.2.2", "3.3.3.3", "4.4.4.4"},
}, structs.AssignServiceManualVIPsResponse{
Found: false,
})
}

View File

@ -794,9 +794,13 @@ func (c *FSM) applyManualVirtualIPs(buf []byte, index uint64) interface{} {
panic(fmt.Errorf("failed to decode request: %v", err))
}
if err := c.state.AssignManualVirtualIPs(index, req.Service, req.ManualIPs); err != nil {
found, unassignedFrom, err := c.state.AssignManualServiceVIPs(index, req.Service, req.ManualIPs)
if err != nil {
c.logger.Warn("AssignManualVirtualIPs failed", "error", err)
return err
}
return nil
return structs.AssignServiceManualVIPsResponse{
Found: found,
UnassignedFrom: unassignedFrom,
}
}

View File

@ -5,6 +5,7 @@ package consul
import (
"fmt"
"net"
"github.com/hashicorp/go-bexpr"
"github.com/hashicorp/go-hclog"
@ -17,6 +18,8 @@ import (
"github.com/hashicorp/consul/agent/structs"
)
const MaximumManualVIPsPerService = 8
// Internal endpoint is used to query the miscellaneous info that
// does not necessarily fit into the other systems. It is also
// used to hold undocumented APIs that users should not rely on.
@ -741,6 +744,55 @@ func (m *Internal) PeeredUpstreams(args *structs.PartitionSpecificRequest, reply
})
}
// AssignManualServiceVIPs allows for assigning virtual IPs to a service manually, so that they can
// be returned along with discovery chain information for use by transparent proxies.
func (m *Internal) AssignManualServiceVIPs(args *structs.AssignServiceManualVIPsRequest, reply *structs.AssignServiceManualVIPsResponse) error {
if done, err := m.srv.ForwardRPC("Internal.AssignManualServiceVIPs", args, reply); done {
return err
}
var authzCtx acl.AuthorizerContext
authz, err := m.srv.ResolveTokenAndDefaultMeta(args.Token, &args.EnterpriseMeta, &authzCtx)
if err != nil {
return err
}
if err := authz.ToAllowAuthorizer().MeshWriteAllowed(&authzCtx); err != nil {
return err
}
if err := m.srv.validateEnterpriseRequest(&args.EnterpriseMeta, true); err != nil {
return err
}
if len(args.ManualVIPs) > MaximumManualVIPsPerService {
return fmt.Errorf("cannot associate more than %d manual virtual IPs with the same service", MaximumManualVIPsPerService)
}
for _, ip := range args.ManualVIPs {
parsedIP := net.ParseIP(ip)
if parsedIP == nil || parsedIP.To4() == nil {
return fmt.Errorf("%q is not a valid IPv4 address", parsedIP.String())
}
}
req := state.ServiceVirtualIP{
Service: structs.PeeredServiceName{
ServiceName: structs.NewServiceName(args.Service, &args.EnterpriseMeta),
},
ManualIPs: args.ManualVIPs,
}
resp, err := m.srv.raftApplyMsgpack(structs.UpdateVirtualIPRequestType, req)
if err != nil {
return err
}
typedResp, ok := resp.(structs.AssignServiceManualVIPsResponse)
if !ok {
return fmt.Errorf("unexpected type %T for AssignManualServiceVIPs", resp)
}
*reply = typedResp
return nil
}
// EventFire is a bit of an odd endpoint, but it allows for a cross-DC RPC
// call to fire an event. The primary use case is to enable user events being
// triggered in a remote DC.

View File

@ -3621,3 +3621,109 @@ func testUUID() string {
buf[8:10],
buf[10:16])
}
func TestInternal_AssignManualServiceVIPs(t *testing.T) {
if testing.Short() {
t.Skip("too slow for testing.Short")
}
t.Parallel()
dir1, s1 := testServer(t)
defer os.RemoveAll(dir1)
defer s1.Shutdown()
codec := rpcClient(t, s1)
defer codec.Close()
testrpc.WaitForLeader(t, s1.RPC, "dc1")
// Set up web service with no manual IPs, and an existing service with manual IPs set.
registerReq := structs.RegisterRequest{
Datacenter: "dc1",
Node: "foo",
Address: "127.0.0.1",
Service: &structs.NodeService{
ID: "web",
Service: "web",
Port: 8888,
Connect: structs.ServiceConnect{Native: true},
},
}
var out struct{}
require.NoError(t, msgpackrpc.CallWithCodec(codec, "Catalog.Register", &registerReq, &out))
registerReq2 := structs.RegisterRequest{
Datacenter: "dc1",
Node: "foo",
Address: "127.0.0.1",
Service: &structs.NodeService{
ID: "existing",
Service: "existing",
Port: 9999,
Connect: structs.ServiceConnect{Native: true},
},
}
require.NoError(t, msgpackrpc.CallWithCodec(codec, "Catalog.Register", &registerReq2, &out))
req := structs.AssignServiceManualVIPsRequest{
Service: "existing",
ManualVIPs: []string{"8.8.8.8", "9.9.9.9"},
}
var resp structs.AssignServiceManualVIPsResponse
require.NoError(t, msgpackrpc.CallWithCodec(codec, "Internal.AssignManualServiceVIPs", req, &resp))
type testcase struct {
name string
req structs.AssignServiceManualVIPsRequest
expect structs.AssignServiceManualVIPsResponse
expectErr string
}
run := func(t *testing.T, tc testcase) {
var resp structs.AssignServiceManualVIPsResponse
err := msgpackrpc.CallWithCodec(codec, "Internal.AssignManualServiceVIPs", tc.req, &resp)
if tc.expectErr != "" {
require.Error(t, err)
require.Contains(t, err.Error(), tc.expectErr)
return
}
require.Equal(t, tc.expect, resp)
}
tcs := []testcase{
{
name: "successful manual ip assignment",
req: structs.AssignServiceManualVIPsRequest{
Service: "web",
ManualVIPs: []string{"1.1.1.1", "2.2.2.2"},
},
expect: structs.AssignServiceManualVIPsResponse{Found: true},
},
{
name: "reassign existing ip",
req: structs.AssignServiceManualVIPsRequest{
Service: "web",
ManualVIPs: []string{"8.8.8.8"},
},
expect: structs.AssignServiceManualVIPsResponse{
Found: true,
UnassignedFrom: []structs.PeeredServiceName{
{
ServiceName: structs.ServiceNameFromString("existing"),
},
},
},
},
{
name: "invalid ip",
req: structs.AssignServiceManualVIPsRequest{
Service: "web",
ManualVIPs: []string{"3.3.3.3", "invalid"},
},
expect: structs.AssignServiceManualVIPsResponse{},
expectErr: "not a valid",
},
}
for _, tc := range tcs {
t.Run(tc.name, func(t *testing.T) {
run(t, tc)
})
}
}

View File

@ -18,6 +18,7 @@ import (
"github.com/hashicorp/consul/agent/structs"
"github.com/hashicorp/consul/api"
"github.com/hashicorp/consul/lib"
"github.com/hashicorp/consul/lib/maps"
"github.com/hashicorp/consul/types"
)
@ -1090,7 +1091,14 @@ func assignServiceVirtualIP(tx WriteTxn, idx uint64, psn structs.PeeredServiceNa
return result.String(), nil
}
func (s *Store) AssignManualVirtualIPs(idx uint64, psn structs.PeeredServiceName, ips []string) error {
// AssignManualServiceVIPs attempts to associate a list of manual virtual IP addresses with a given service name.
// Any IP addresses given will be removed from other services in the same partition. This is done to ensure
// that a manual VIP can only exist once for a given partition.
// This function returns:
// - a bool indicating whether the given service exists.
// - a list of service names that had ip addresses removed from them.
// - an error indicating success or failure of the call.
func (s *Store) AssignManualServiceVIPs(idx uint64, psn structs.PeeredServiceName, ips []string) (bool, []structs.PeeredServiceName, error) {
tx := s.db.WriteTxn(idx)
defer tx.Abort()
@ -1099,10 +1107,11 @@ func (s *Store) AssignManualVirtualIPs(idx uint64, psn structs.PeeredServiceName
for _, ip := range ips {
assignedIPs[ip] = struct{}{}
}
modifiedEntries := make(map[structs.PeeredServiceName]struct{})
for ip := range assignedIPs {
entry, err := tx.First(tableServiceVirtualIPs, indexManualVIPs, psn.ServiceName.PartitionOrDefault(), ip)
if err != nil {
return fmt.Errorf("failed service virtual IP lookup: %s", err)
return false, nil, fmt.Errorf("failed service virtual IP lookup: %s", err)
}
if entry == nil {
@ -1126,17 +1135,18 @@ func (s *Store) AssignManualVirtualIPs(idx uint64, psn structs.PeeredServiceName
newEntry.ManualIPs = filteredIPs
newEntry.ModifyIndex = idx
if err := tx.Insert(tableServiceVirtualIPs, newEntry); err != nil {
return fmt.Errorf("failed inserting service virtual IP entry: %s", err)
return false, nil, fmt.Errorf("failed inserting service virtual IP entry: %s", err)
}
modifiedEntries[newEntry.Service] = struct{}{}
}
entry, err := tx.First(tableServiceVirtualIPs, indexID, psn)
if err != nil {
return fmt.Errorf("failed service virtual IP lookup: %s", err)
return false, nil, fmt.Errorf("failed service virtual IP lookup: %s", err)
}
if entry == nil {
return nil
return false, nil, nil
}
newEntry := entry.(ServiceVirtualIP)
@ -1144,13 +1154,16 @@ func (s *Store) AssignManualVirtualIPs(idx uint64, psn structs.PeeredServiceName
newEntry.ModifyIndex = idx
if err := tx.Insert(tableServiceVirtualIPs, newEntry); err != nil {
return fmt.Errorf("failed inserting service virtual IP entry: %s", err)
return false, nil, fmt.Errorf("failed inserting service virtual IP entry: %s", err)
}
if err := updateVirtualIPMaxIndexes(tx, idx, psn.ServiceName.PartitionOrDefault(), psn.Peer); err != nil {
return err
return false, nil, err
}
if err = tx.Commit(); err != nil {
return false, nil, err
}
return tx.Commit()
return true, maps.SliceOfKeys(modifiedEntries), nil
}
func updateVirtualIPMaxIndexes(txn WriteTxn, idx uint64, partition, peerName string) error {

View File

@ -1964,8 +1964,11 @@ func TestStateStore_AssignManualVirtualIPs(t *testing.T) {
setVirtualIPFlags(t, s)
// Attempt to assign manual virtual IPs to a service that doesn't exist - should be a no-op.
psn := structs.PeeredServiceName{ServiceName: structs.ServiceName{Name: "foo"}}
require.NoError(t, s.AssignManualVirtualIPs(0, psn, []string{"7.7.7.7", "8.8.8.8"}))
psn := structs.PeeredServiceName{ServiceName: structs.ServiceName{Name: "foo", EnterpriseMeta: *acl.DefaultEnterpriseMeta()}}
found, svcs, err := s.AssignManualServiceVIPs(0, psn, []string{"7.7.7.7", "8.8.8.8"})
require.NoError(t, err)
require.False(t, found)
require.Empty(t, svcs)
serviceVIP, err := s.ServiceManualVIPs(psn)
require.NoError(t, err)
require.Nil(t, serviceVIP)
@ -1997,24 +2000,20 @@ func TestStateStore_AssignManualVirtualIPs(t *testing.T) {
require.Empty(t, serviceVIP.ManualIPs)
// Attempt to assign manual virtual IPs again.
require.NoError(t, s.AssignManualVirtualIPs(2, psn, []string{"7.7.7.7", "8.8.8.8"}))
found, svcs, err = s.AssignManualServiceVIPs(2, psn, []string{"7.7.7.7", "8.8.8.8"})
require.NoError(t, err)
require.True(t, found)
require.Empty(t, svcs)
serviceVIP, err = s.ServiceManualVIPs(psn)
require.NoError(t, err)
require.Equal(t, "0.0.0.1", serviceVIP.IP.String())
require.Equal(t, serviceVIP.ManualIPs, []string{"7.7.7.7", "8.8.8.8"})
// Register another service
ns2 := &structs.NodeService{
ID: "bar",
Service: "bar",
Address: "2.2.2.2",
Port: 2222,
Connect: structs.ServiceConnect{Native: true},
EnterpriseMeta: *entMeta,
}
// Service successfully registers into the state store.
require.NoError(t, s.EnsureService(3, "node1", ns2))
// Register another service via config entry.
s.EnsureConfigEntry(3, &structs.ServiceResolverConfigEntry{
Kind: structs.ServiceResolver,
Name: "bar",
})
psn2 := structs.PeeredServiceName{ServiceName: structs.ServiceName{Name: "bar"}}
vip, err = s.VirtualIPForService(psn2)
@ -2023,7 +2022,10 @@ func TestStateStore_AssignManualVirtualIPs(t *testing.T) {
// Attempt to assign manual virtual IPs for bar, with one IP overlapping with foo.
// This should cause the ip to be removed from foo's list of manual IPs.
require.NoError(t, s.AssignManualVirtualIPs(4, psn2, []string{"7.7.7.7", "9.9.9.9"}))
found, svcs, err = s.AssignManualServiceVIPs(4, psn2, []string{"7.7.7.7", "9.9.9.9"})
require.NoError(t, err)
require.True(t, found)
require.ElementsMatch(t, svcs, []structs.PeeredServiceName{psn})
serviceVIP, err = s.ServiceManualVIPs(psn)
require.NoError(t, err)

View File

@ -100,6 +100,7 @@ func init() {
registerEndpoint("/v1/internal/ui/gateway-intentions/", []string{"GET"}, (*HTTPHandlers).UIGatewayIntentions)
registerEndpoint("/v1/internal/ui/service-topology/", []string{"GET"}, (*HTTPHandlers).UIServiceTopology)
registerEndpoint("/v1/internal/acl/authorize", []string{"POST"}, (*HTTPHandlers).ACLAuthorize)
registerEndpoint("/v1/internal/service-virtual-ip", []string{"PUT"}, (*HTTPHandlers).AssignManualServiceVIPs)
registerEndpoint("/v1/kv/", []string{"GET", "PUT", "DELETE"}, (*HTTPHandlers).KVSEndpoint)
registerEndpoint("/v1/operator/raft/configuration", []string{"GET"}, (*HTTPHandlers).OperatorRaftConfiguration)
registerEndpoint("/v1/operator/raft/transfer-leader", []string{"POST"}, (*HTTPHandlers).OperatorRaftTransferLeader)

View File

@ -59,3 +59,15 @@ func (h *HealthSummary) Add(status string) {
h.Critical++
}
}
type AssignServiceManualVIPsRequest struct {
Service string
ManualVIPs []string
DCSpecificRequest
}
type AssignServiceManualVIPsResponse struct {
Found bool
UnassignedFrom []PeeredServiceName
}

64
api/internal.go Normal file
View File

@ -0,0 +1,64 @@
package api
import "context"
// Internal can be used to query endpoints that are intended for
// Hashicorp internal-use only.
type Internal struct {
c *Client
}
// Internal returns a handle to endpoints that are for internal
// Hashicorp usage only. There is not guarantee that these will
// be backwards-compatible or supported, so usage of these is
// not encouraged.
func (c *Client) Internal() *Internal {
return &Internal{c}
}
type AssignServiceManualVIPsRequest struct {
Service string
ManualVIPs []string
}
type AssignServiceManualVIPsResponse struct {
ServiceFound bool `json:"Found"`
UnassignedFrom []PeeredServiceName
}
type PeeredServiceName struct {
ServiceName CompoundServiceName
Peer string
}
func (i *Internal) AssignServiceVirtualIP(
ctx context.Context,
service string,
manualVIPs []string,
wo *WriteOptions,
) (*AssignServiceManualVIPsResponse, *QueryMeta, error) {
req := i.c.newRequest("PUT", "/v1/internal/service-virtual-ip")
req.setWriteOptions(wo)
req.ctx = ctx
req.obj = AssignServiceManualVIPsRequest{
Service: service,
ManualVIPs: manualVIPs,
}
rtt, resp, err := i.c.doRequest(req)
if err != nil {
return nil, nil, err
}
defer closeResponseBody(resp)
if err := requireOK(resp); err != nil {
return nil, nil, err
}
qm := &QueryMeta{RequestTime: rtt}
parseQueryMeta(resp, qm)
var out AssignServiceManualVIPsResponse
if err := decodeBody(resp, &out); err != nil {
return nil, nil, err
}
return &out, qm, nil
}

114
api/internal_test.go Normal file
View File

@ -0,0 +1,114 @@
package api
import (
"context"
"testing"
"time"
"github.com/stretchr/testify/require"
)
func TestAPI_Internal_AssignServiceVirtualIP(t *testing.T) {
t.Parallel()
doTest_Internal_AssignServiceVirtualIP(t, &WriteOptions{
Namespace: defaultNamespace,
Partition: defaultPartition,
})
}
func doTest_Internal_AssignServiceVirtualIP(t *testing.T, writeOpts *WriteOptions) {
c, s := makeClient(t)
defer s.Stop()
ctx, cancel := context.WithTimeout(context.Background(), 20*time.Second)
defer cancel()
if writeOpts.Partition != "" {
_, _, err := c.Partitions().Create(ctx, &Partition{Name: writeOpts.Partition}, nil)
require.NoError(t, err)
}
if writeOpts.Namespace != "" {
_, _, err := c.Namespaces().Create(&Namespace{Name: writeOpts.Namespace, Partition: writeOpts.Partition}, nil)
require.NoError(t, err)
}
// Create resolvers that we can attach VIPs to.
for _, name := range []string{"one", "two", "three"} {
ok, _, err := c.ConfigEntries().Set(&ServiceResolverConfigEntry{
Kind: ServiceResolver,
Name: name,
Namespace: writeOpts.Namespace,
Partition: writeOpts.Partition,
}, writeOpts)
require.NoError(t, err)
require.True(t, ok)
}
tests := []struct {
tName string
svcName string
vips []string
expectFound bool
expectUnassignedFrom []PeeredServiceName
}{
{
tName: "missing service is no-op",
svcName: "missing",
vips: []string{"1.1.1.1", "2.2.2.2"},
expectFound: false,
expectUnassignedFrom: nil,
},
{
tName: "set vips for one",
svcName: "one",
vips: []string{"1.1.1.1", "2.2.2.2", "3.3.3.3"},
expectFound: true,
expectUnassignedFrom: nil,
},
{
tName: "move vip to two",
svcName: "two",
vips: []string{"2.2.2.2"},
expectFound: true,
expectUnassignedFrom: []PeeredServiceName{
{ServiceName: CompoundServiceName{Name: "one", Namespace: writeOpts.Namespace, Partition: writeOpts.Partition}},
},
},
{
tName: "move vip to three",
svcName: "three",
vips: []string{"3.3.3.3"},
expectFound: true,
expectUnassignedFrom: []PeeredServiceName{
{ServiceName: CompoundServiceName{Name: "one", Namespace: writeOpts.Namespace, Partition: writeOpts.Partition}},
},
},
{
tName: "no-op try move vips to missing",
svcName: "missing",
vips: []string{"1.1.1.1", "2.2.2.2", "3.3.3.3"},
expectFound: false,
expectUnassignedFrom: nil,
},
{
tName: "move all vips back to one",
svcName: "one",
vips: []string{"1.1.1.1", "2.2.2.2", "3.3.3.3", "4.4.4.4"},
expectFound: true,
expectUnassignedFrom: []PeeredServiceName{
{ServiceName: CompoundServiceName{Name: "two", Namespace: writeOpts.Namespace, Partition: writeOpts.Partition}},
{ServiceName: CompoundServiceName{Name: "three", Namespace: writeOpts.Namespace, Partition: writeOpts.Partition}},
},
},
}
internal := c.Internal()
for _, tc := range tests {
t.Run(tc.tName, func(t *testing.T) {
resp, _, err := internal.AssignServiceVirtualIP(ctx, tc.svcName, tc.vips, writeOpts)
require.NoError(t, err)
require.Equal(t, tc.expectFound, resp.ServiceFound)
require.ElementsMatch(t, tc.expectUnassignedFrom, resp.UnassignedFrom)
})
}
}