Browse Source

Add traffic permissions integration tests. (#19008)

Add traffic permissions integration tests.
pull/19104/head
Eric Haberkorn 1 year ago committed by GitHub
parent
commit
ad3aab1ef7
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
  1. 10
      .github/workflows/test-integrations.yml
  2. 4
      Makefile
  3. 12
      internal/resource/resourcetest/builder.go
  4. 29
      internal/resource/resourcetest/client.go
  5. 27
      test/integration/consul-container/assets/Dockerfile-consul-dataplane
  6. 5
      test/integration/consul-container/libs/cluster/builder.go
  7. 79
      test/integration/consul-container/libs/cluster/dataplane.go
  8. 3
      test/integration/consul-container/libs/topology/peering_topology.go
  9. 2
      test/integration/consul-container/test/multiport/explicit_destination_test.go
  10. 657
      test/integration/consul-container/test/trafficpermissions/tcp_test.go

10
.github/workflows/test-integrations.yml

@ -116,7 +116,7 @@ jobs:
--packages="./command/agent/consul" \
--junitfile $TEST_RESULTS_DIR/results.xml -- \
-run TestConsul
# NOTE: ENT specific step as we store secrets in Vault.
- name: Authenticate to Vault
if: ${{ endsWith(github.repository, '-enterprise') }}
@ -257,7 +257,7 @@ jobs:
- name: Generate Envoy Job Matrix
id: set-matrix
env:
# this is further going to multiplied in envoy-integration tests by the
# this is further going to multiplied in envoy-integration tests by the
# other dimensions in the matrix. Currently TOTAL_RUNNERS would be
# multiplied by 8 based on these values:
# envoy-version: ["1.24.10", "1.25.9", "1.26.4", "1.27.0"]
@ -281,7 +281,7 @@ jobs:
| jq --raw-input --argjson runnercount "$NUM_RUNNERS" "$JQ_SLICER" \
| jq --compact-output 'map(join("|"))'
} >> "$GITHUB_OUTPUT"
envoy-integration-test:
runs-on: ${{ fromJSON(needs.setup.outputs.compute-large) }}
needs:
@ -384,7 +384,7 @@ jobs:
contents: read
env:
ENVOY_VERSION: "1.25.4"
CONSUL_DATAPLANE_IMAGE: "docker.io/hashicorppreview/consul-dataplane:1.3-dev"
CONSUL_DATAPLANE_IMAGE: "docker.io/hashicorppreview/consul-dataplane:1.3-dev-ubi"
steps:
- uses: actions/checkout@c85c95e3d7251135ab7dc9ce3241c5835cc595a9 # v3.5.3
# NOTE: This step is specifically needed for ENT. It allows us to access the required private HashiCorp repos.
@ -417,7 +417,7 @@ jobs:
if: steps.buildConsulEnvoyImage.outcome == 'failure'
run: docker build -t consul-envoy:target-version --build-arg CONSUL_IMAGE=${{ env.CONSUL_LATEST_IMAGE_NAME }}:local --build-arg ENVOY_VERSION=${{ env.ENVOY_VERSION }} -f ./test/integration/consul-container/assets/Dockerfile-consul-envoy ./test/integration/consul-container/assets
- name: Build consul-dataplane:local image
run: docker build -t consul-dataplane:local --build-arg CONSUL_DATAPLANE_IMAGE=${{ env.CONSUL_DATAPLANE_IMAGE }} -f ./test/integration/consul-container/assets/Dockerfile-consul-dataplane ./test/integration/consul-container/assets
run: docker build -t consul-dataplane:local --build-arg CONSUL_IMAGE=${{ env.CONSUL_LATEST_IMAGE_NAME }}:local --build-arg CONSUL_DATAPLANE_IMAGE=${{ env.CONSUL_DATAPLANE_IMAGE }} -f ./test/integration/consul-container/assets/Dockerfile-consul-dataplane ./test/integration/consul-container/assets
- name: Configure GH workaround for ipv6 loopback
if: ${{ !endsWith(github.repository, '-enterprise') }}
run: |

4
Makefile

@ -66,7 +66,7 @@ UI_BUILD_TAG?=consul-build-ui
BUILD_CONTAINER_NAME?=consul-builder
CONSUL_IMAGE_VERSION?=latest
ENVOY_VERSION?='1.25.4'
CONSUL_DATAPLANE_IMAGE := $(or $(CONSUL_DATAPLANE_IMAGE),"docker.io/hashicorppreview/consul-dataplane:1.3-dev")
CONSUL_DATAPLANE_IMAGE := $(or $(CONSUL_DATAPLANE_IMAGE),"docker.io/hashicorppreview/consul-dataplane:1.3-dev-ubi")
CONSUL_VERSION?=$(shell cat version/VERSION)
@ -349,7 +349,7 @@ test-compat-integ-setup: dev-docker
@docker run --rm -t $(CONSUL_COMPAT_TEST_IMAGE):local consul version
@# 'consul-envoy:target-version' is needed by compatibility integ test
@docker build -t consul-envoy:target-version --build-arg CONSUL_IMAGE=$(CONSUL_COMPAT_TEST_IMAGE):local --build-arg ENVOY_VERSION=${ENVOY_VERSION} -f ./test/integration/consul-container/assets/Dockerfile-consul-envoy ./test/integration/consul-container/assets
@docker build -t consul-dataplane:local --build-arg CONSUL_DATAPLANE_IMAGE=${CONSUL_DATAPLANE_IMAGE} -f ./test/integration/consul-container/assets/Dockerfile-consul-dataplane ./test/integration/consul-container/assets
@docker build -t consul-dataplane:local --build-arg CONSUL_IMAGE=$(CONSUL_COMPAT_TEST_IMAGE):local --build-arg CONSUL_DATAPLANE_IMAGE=${CONSUL_DATAPLANE_IMAGE} -f ./test/integration/consul-container/assets/Dockerfile-consul-dataplane ./test/integration/consul-container/assets
.PHONY: test-compat-integ
test-compat-integ: test-compat-integ-setup ## Test compat integ

12
internal/resource/resourcetest/builder.go

@ -4,6 +4,7 @@
package resourcetest
import (
"context"
"strings"
"github.com/oklog/ulid/v2"
@ -134,7 +135,14 @@ func (b *resourceBuilder) ReferenceNoSection() *pbresource.Reference {
func (b *resourceBuilder) Write(t T, client pbresource.ResourceServiceClient) *pbresource.Resource {
t.Helper()
ctx := testutil.TestContext(t)
var ctx context.Context
rtestClient, ok := client.(*Client)
if ok {
ctx = rtestClient.Context(t)
} else {
ctx = testutil.TestContext(t)
rtestClient = NewClient(client)
}
res := b.resource
@ -170,7 +178,7 @@ func (b *resourceBuilder) Write(t T, client pbresource.ResourceServiceClient) *p
id := proto.Clone(rsp.Resource.Id).(*pbresource.ID)
id.Uid = ""
t.Cleanup(func() {
NewClient(client).MustDelete(t, id)
rtestClient.MustDelete(t, id)
})
}

29
internal/resource/resourcetest/client.go

@ -4,6 +4,7 @@
package resourcetest
import (
"context"
"fmt"
"math/rand"
"time"
@ -11,6 +12,7 @@ import (
"github.com/stretchr/testify/require"
"golang.org/x/exp/slices"
"google.golang.org/grpc/codes"
"google.golang.org/grpc/metadata"
"google.golang.org/grpc/status"
"github.com/hashicorp/consul/internal/resource"
@ -24,13 +26,19 @@ type Client struct {
timeout time.Duration
wait time.Duration
token string
}
func NewClient(client pbresource.ResourceServiceClient) *Client {
return NewClientWithACLToken(client, "")
}
func NewClientWithACLToken(client pbresource.ResourceServiceClient, token string) *Client {
return &Client{
ResourceServiceClient: client,
timeout: 7 * time.Second,
wait: 25 * time.Millisecond,
token: token,
}
}
@ -46,7 +54,7 @@ func (client *Client) retry(t T, fn func(r *retry.R)) {
}
func (client *Client) PublishResources(t T, resources []*pbresource.Resource) {
ctx := testutil.TestContext(t)
ctx := client.Context(t)
// Randomize the order of insertion. Generally insertion order shouldn't matter as the
// controllers should eventually converge on the desired state. The exception to this
@ -111,10 +119,23 @@ func (client *Client) PublishResources(t T, resources []*pbresource.Resource) {
require.Empty(t, resources, "Could not publish all resources - some resources have invalid owner references")
}
func (client *Client) Context(t T) context.Context {
ctx := testutil.TestContext(t)
if client.token != "" {
md := metadata.New(map[string]string{
"x-consul-token": client.token,
})
ctx = metadata.NewOutgoingContext(ctx, md)
}
return ctx
}
func (client *Client) RequireResourceNotFound(t T, id *pbresource.ID) {
t.Helper()
rsp, err := client.Read(testutil.TestContext(t), &pbresource.ReadRequest{Id: id})
rsp, err := client.Read(client.Context(t), &pbresource.ReadRequest{Id: id})
require.Error(t, err)
require.Equal(t, codes.NotFound, status.Code(err))
require.Nil(t, rsp)
@ -123,7 +144,7 @@ func (client *Client) RequireResourceNotFound(t T, id *pbresource.ID) {
func (client *Client) RequireResourceExists(t T, id *pbresource.ID) *pbresource.Resource {
t.Helper()
rsp, err := client.Read(testutil.TestContext(t), &pbresource.ReadRequest{Id: id})
rsp, err := client.Read(client.Context(t), &pbresource.ReadRequest{Id: id})
require.NoError(t, err, "error reading %s with type %s", id.Name, resource.ToGVK(id.Type))
require.NotNil(t, rsp)
return rsp.Resource
@ -261,7 +282,7 @@ func (client *Client) ResolveResourceID(t T, id *pbresource.ID) *pbresource.ID {
func (client *Client) MustDelete(t T, id *pbresource.ID) {
t.Helper()
ctx := testutil.TestContext(t)
ctx := client.Context(t)
client.retry(t, func(r *retry.R) {
_, err := client.Delete(ctx, &pbresource.DeleteRequest{Id: id})

27
test/integration/consul-container/assets/Dockerfile-consul-dataplane

@ -2,7 +2,30 @@
# SPDX-License-Identifier: BUSL-1.1
ARG CONSUL_DATAPLANE_IMAGE
ARG CONSUL_IMAGE
# Docker doesn't support expansion in COPY --copy, so we need to create an intermediate image.
FROM ${CONSUL_IMAGE} as consul
FROM ${CONSUL_DATAPLANE_IMAGE} as consuldataplane
COPY --from=busybox:uclibc /bin/sh /bin/sh
COPY --from=ghcr.io/tarampampam/curl:latest /bin/curl /bin/curl
USER root
# On Mac M1s when TProxy is enabled, consul-dataplane that are spawned from this image
# (only used in consul-container integration tests) will terminate with the below error.
# It is related to tproxy-startup.sh calling iptables SDK which then calls the underly
# iptables. We are investigating how this works on M1s with consul-envoy images which
# do not have this problem. For the time being tproxy tests on Mac M1s will fail locally
# but pass in CI.
#
# Error setting up traffic redirection rules: failed to run command: /sbin/iptables -t nat -N CONSUL_PROXY_INBOUND, err: exit status 1, output: iptables: Failed to initialize nft: Protocol not supported
RUN microdnf install -y iptables sudo nc \
&& usermod -a -G wheel consul-dataplane \
&& echo 'consul-dataplane ALL=(ALL) NOPASSWD:ALL' >> /etc/sudoers
COPY --from=consul /bin/consul /bin/consul
COPY tproxy-startup.sh /bin/tproxy-startup.sh
RUN chmod +x /bin/tproxy-startup.sh && chown root:root /bin/tproxy-startup.sh
USER 100

5
test/integration/consul-container/libs/cluster/builder.go

@ -273,6 +273,11 @@ func (b *Builder) Peering(enable bool) *Builder {
return b
}
func (b *Builder) SetACLToken(token string) *Builder {
b.conf.Set("acl.tokens.agent", token)
return b
}
func (b *Builder) NodeID(nodeID string) *Builder {
b.conf.Set("node_id", nodeID)
return b

79
test/integration/consul-container/libs/cluster/dataplane.go

@ -6,11 +6,14 @@ package cluster
import (
"context"
"fmt"
"io"
"strconv"
"strings"
"time"
"github.com/hashicorp/consul/test/integration/consul-container/libs/utils"
"github.com/testcontainers/testcontainers-go"
"github.com/testcontainers/testcontainers-go/wait"
"strconv"
"time"
)
type ConsulDataplaneContainer struct {
@ -27,6 +30,10 @@ func (g ConsulDataplaneContainer) GetAddr() (string, int) {
return g.ip, g.appPort[0]
}
func (g ConsulDataplaneContainer) GetServiceName() string {
return g.serviceName
}
// GetAdminAddr returns the external admin port
func (g ConsulDataplaneContainer) GetAdminAddr() (string, int) {
return "localhost", g.externalAdminPort
@ -36,13 +43,28 @@ func (c ConsulDataplaneContainer) Terminate() error {
return TerminateContainer(c.ctx, c.container, true)
}
func (g ConsulDataplaneContainer) Exec(ctx context.Context, cmd []string) (string, error) {
exitCode, reader, err := g.container.Exec(ctx, cmd)
if err != nil {
return "", fmt.Errorf("exec with error %s", err)
}
if exitCode != 0 {
return "", fmt.Errorf("exec with exit code %d", exitCode)
}
buf, err := io.ReadAll(reader)
if err != nil {
return "", fmt.Errorf("error reading from exec output: %w", err)
}
return string(buf), nil
}
func (g ConsulDataplaneContainer) GetStatus() (string, error) {
state, err := g.container.State(g.ctx)
return state.Status, err
}
func NewConsulDataplane(ctx context.Context, proxyID string, serverAddresses string, grpcPort int, serviceBindPorts []int,
node Agent, containerArgs ...string) (*ConsulDataplaneContainer, error) {
node Agent, tproxy bool, bootstrapToken string, containerArgs ...string) (*ConsulDataplaneContainer, error) {
namePrefix := fmt.Sprintf("%s-consul-dataplane-%s", node.GetDatacenter(), proxyID)
containerName := utils.RandName(namePrefix)
@ -70,7 +92,39 @@ func NewConsulDataplane(ctx context.Context, proxyID string, serverAddresses str
copy(exposedPorts, appPortStrs)
exposedPorts = append(exposedPorts, adminPortStr)
command := []string{
req := testcontainers.ContainerRequest{
Image: "consul-dataplane:local",
WaitingFor: wait.ForLog("").WithStartupTimeout(60 * time.Second),
AutoRemove: false,
Name: containerName,
Env: map[string]string{},
}
var command []string
if tproxy {
req.Entrypoint = []string{"sh", "/bin/tproxy-startup.sh"}
req.Env["REDIRECT_TRAFFIC_ARGS"] = strings.Join(
[]string{
// TODO once we run this on a different pod from Consul agents, we can eliminate most of this.
"-exclude-inbound-port", fmt.Sprint(internalAdminPort),
"-exclude-inbound-port", "8300",
"-exclude-inbound-port", "8301",
"-exclude-inbound-port", "8302",
"-exclude-inbound-port", "8500",
"-exclude-inbound-port", "8502",
"-exclude-inbound-port", "8600",
"-proxy-inbound-port", "20000",
"-consul-dns-ip", "127.0.0.1",
"-consul-dns-port", "8600",
},
" ",
)
req.CapAdd = append(req.CapAdd, "NET_ADMIN")
command = append(command, "consul-dataplane")
}
command = append(command,
"-addresses", serverAddresses,
fmt.Sprintf("-grpc-port=%d", grpcPort),
fmt.Sprintf("-proxy-id=%s", proxyID),
@ -81,19 +135,16 @@ func NewConsulDataplane(ctx context.Context, proxyID string, serverAddresses str
"-envoy-concurrency=2",
"-tls-disabled",
fmt.Sprintf("-envoy-admin-bind-port=%d", internalAdminPort),
}
command = append(command, containerArgs...)
)
req := testcontainers.ContainerRequest{
Image: "consul-dataplane:local",
WaitingFor: wait.ForLog("").WithStartupTimeout(60 * time.Second),
AutoRemove: false,
Name: containerName,
Cmd: command,
Env: map[string]string{},
if bootstrapToken != "" {
command = append(command,
"-credential-type=static",
fmt.Sprintf("-static-token=%s", bootstrapToken))
}
req.Cmd = append(command, containerArgs...)
info, err := LaunchContainerOnNode(ctx, node, req, exposedPorts)
if err != nil {
return nil, err

3
test/integration/consul-container/libs/topology/peering_topology.go

@ -272,6 +272,9 @@ func NewClusterWithConfig(
Client().
Peering(true).
RetryJoin(retryJoin...)
if cluster.TokenBootstrap != "" {
configBuilder.SetACLToken(cluster.TokenBootstrap)
}
clientConf := configBuilder.ToAgentConfig(t)
t.Logf("%s client config: \n%s", opts.Datacenter, clientConf.JSON)
if clientHclConfig != "" {

2
test/integration/consul-container/test/multiport/explicit_destination_test.go

@ -87,7 +87,7 @@ func createServiceAndDataplane(t *testing.T, node libcluster.Agent, proxyID, ser
})
// Create Consul Dataplane
dp, err := libcluster.NewConsulDataplane(context.Background(), proxyID, "0.0.0.0", 8502, serviceBindPorts, node)
dp, err := libcluster.NewConsulDataplane(context.Background(), proxyID, "0.0.0.0", 8502, serviceBindPorts, node, false, "")
require.NoError(t, err)
deferClean.Add(func() {
_ = dp.Terminate()

657
test/integration/consul-container/test/trafficpermissions/tcp_test.go

@ -0,0 +1,657 @@
// Copyright (c) HashiCorp, Inc.
// SPDX-License-Identifier: BUSL-1.1
package trafficpermissions
import (
"context"
"fmt"
"strings"
"testing"
"github.com/hashicorp/consul/sdk/testutil/retry"
rtest "github.com/hashicorp/consul/internal/resource/resourcetest"
pbauth "github.com/hashicorp/consul/proto-public/pbauth/v2beta1"
pbcatalog "github.com/hashicorp/consul/proto-public/pbcatalog/v2beta1"
pbmesh "github.com/hashicorp/consul/proto-public/pbmesh/v2beta1"
"github.com/hashicorp/consul/proto-public/pbresource"
libcluster "github.com/hashicorp/consul/test/integration/consul-container/libs/cluster"
libservice "github.com/hashicorp/consul/test/integration/consul-container/libs/service"
"github.com/hashicorp/consul/test/integration/consul-container/libs/topology"
"github.com/hashicorp/consul/test/integration/consul-container/libs/utils"
"github.com/stretchr/testify/require"
)
const (
echoPort = 9999
tcpPort = 8888
staticServerVIP = "240.0.0.1"
staticServerReturnValue = "static-server"
staticServerIdentity = "static-server-identity"
)
type trafficPermissionsCase struct {
tp1 *pbauth.TrafficPermissions
tp2 *pbauth.TrafficPermissions
client1TCPSuccess bool
client1EchoSuccess bool
client2TCPSuccess bool
client2EchoSuccess bool
}
// We are using tproxy to test traffic permissions now because explicitly specifying destinations
// doesn't work when multiple downstreams specify the same destination yet. In the future, we will need
// to update this to use explicit destinations once we infer tproxy destinations from traffic permissions.
//
// This also explicitly uses virtual IPs and virtual ports because Consul DNS doesn't support v2 resources yet.
// We should update this to use Consul DNS when it is working.
func runTrafficPermissionsTests(t *testing.T, aclsEnabled bool, cases map[string]trafficPermissionsCase) {
t.Parallel()
cluster, resourceClient := createCluster(t, aclsEnabled)
serverDataplane := createServerResources(t, resourceClient, cluster, cluster.Agents[1])
client1Dataplane := createClientResources(t, resourceClient, cluster, cluster.Agents[2], 1)
client2Dataplane := createClientResources(t, resourceClient, cluster, cluster.Agents[3], 2)
assertDataplaneContainerState(t, client1Dataplane, "running")
assertDataplaneContainerState(t, client2Dataplane, "running")
assertDataplaneContainerState(t, serverDataplane, "running")
for n, tc := range cases {
t.Run(n, func(t *testing.T) {
storeStaticServerTrafficPermissions(t, resourceClient, tc.tp1, 1)
storeStaticServerTrafficPermissions(t, resourceClient, tc.tp2, 2)
// We must establish a new TCP connection each time because TCP traffic permissions are
// enforced at the connection level.
retry.Run(t, func(r *retry.R) {
assertPassing(r, httpRequestToVirtualAddress, client1Dataplane, tc.client1TCPSuccess)
assertPassing(r, echoToVirtualAddress, client1Dataplane, tc.client1EchoSuccess)
assertPassing(r, httpRequestToVirtualAddress, client2Dataplane, tc.client2TCPSuccess)
assertPassing(r, echoToVirtualAddress, client2Dataplane, tc.client2EchoSuccess)
})
})
}
}
func TestTrafficPermission_TCP_DefaultDeny(t *testing.T) {
cases := map[string]trafficPermissionsCase{
"default deny": {
tp1: nil,
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
"allow everything": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
// IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: true,
client2TCPSuccess: true,
client2EchoSuccess: true,
},
"allow tcp": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
// IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
DestinationRules: []*pbauth.DestinationRule{
{
PortNames: []string{"tcp"},
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: false,
client2TCPSuccess: true,
client2EchoSuccess: false,
},
"client 1 only": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: true,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
"allow all exclude client 1": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
Namespace: "default",
Partition: "default",
Peer: "local",
Exclude: []*pbauth.ExcludeSource{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
},
},
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: true,
client2EchoSuccess: true,
},
"deny takes precedence over allow": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_DENY,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
tp2: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
"deny all exclude service + allow on that service": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_DENY,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
Namespace: "default",
Partition: "default",
Peer: "local",
Exclude: []*pbauth.ExcludeSource{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
},
},
tp2: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
IdentityName: "static-client-1-identity",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: true,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
}
runTrafficPermissionsTests(t, true, cases)
}
func TestTrafficPermission_TCP_DefaultAllow(t *testing.T) {
cases := map[string]trafficPermissionsCase{
"default allow": {
tp1: nil,
client1TCPSuccess: true,
client1EchoSuccess: true,
client2TCPSuccess: true,
client2EchoSuccess: true,
},
"empty allow denies everything": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
},
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
"empty deny denies everything": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_DENY,
},
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
"allow everything": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: true,
client2TCPSuccess: true,
client2EchoSuccess: true,
},
"allow one protocol denies the other protocol": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
DestinationRules: []*pbauth.DestinationRule{
{
PortNames: []string{"tcp"},
},
},
},
},
},
client1TCPSuccess: true,
client1EchoSuccess: false,
client2TCPSuccess: true,
client2EchoSuccess: false,
},
"allow something unrelated": {
tp1: &pbauth.TrafficPermissions{
Destination: &pbauth.Destination{
IdentityName: staticServerIdentity,
},
Action: pbauth.Action_ACTION_ALLOW,
Permissions: []*pbauth.Permission{
{
Sources: []*pbauth.Source{
{
IdentityName: "something-else",
Namespace: "default",
Partition: "default",
Peer: "local",
},
},
},
},
},
client1TCPSuccess: false,
client1EchoSuccess: false,
client2TCPSuccess: false,
client2EchoSuccess: false,
},
}
runTrafficPermissionsTests(t, false, cases)
}
func createServiceAndDataplane(t *testing.T, node libcluster.Agent, cluster *libcluster.Cluster, proxyID, serviceName string, httpPort, grpcPort int, serviceBindPorts []int) (*libcluster.ConsulDataplaneContainer, error) {
leader, err := cluster.Leader()
require.NoError(t, err)
leaderIP := leader.GetIP()
token := cluster.TokenBootstrap
// Do some trickery to ensure that partial completion is correctly torn
// down, but successful execution is not.
var deferClean utils.ResettableDefer
defer deferClean.Execute()
// Create a service and proxy instance
svc, err := libservice.NewExampleService(context.Background(), serviceName, httpPort, grpcPort, node)
if err != nil {
return nil, err
}
deferClean.Add(func() {
_ = svc.Terminate()
})
// Create Consul Dataplane
dp, err := libcluster.NewConsulDataplane(context.Background(), proxyID, leaderIP, 8502, serviceBindPorts, node, true, token)
require.NoError(t, err)
deferClean.Add(func() {
_ = dp.Terminate()
})
// disable cleanup functions now that we have an object with a Terminate() function
deferClean.Reset()
return dp, nil
}
func storeStaticServerTrafficPermissions(t *testing.T, resourceClient *rtest.Client, tp *pbauth.TrafficPermissions, i int) {
id := &pbresource.ID{
Name: fmt.Sprintf("static-server-tp-%d", i),
Type: pbauth.TrafficPermissionsType,
}
if tp == nil {
resourceClient.Delete(resourceClient.Context(t), &pbresource.DeleteRequest{
Id: id,
})
} else {
rtest.ResourceID(id).
WithData(t, tp).
Write(t, resourceClient)
}
}
func createServerResources(t *testing.T, resourceClient *rtest.Client, cluster *libcluster.Cluster, node libcluster.Agent) *libcluster.ConsulDataplaneContainer {
rtest.ResourceID(&pbresource.ID{
Name: "static-server-service",
Type: pbcatalog.ServiceType,
}).
WithData(t, &pbcatalog.Service{
Workloads: &pbcatalog.WorkloadSelector{Prefixes: []string{"static-server"}},
Ports: []*pbcatalog.ServicePort{
{
TargetPort: "tcp",
Protocol: pbcatalog.Protocol_PROTOCOL_TCP,
VirtualPort: 8888,
},
{
TargetPort: "echo",
Protocol: pbcatalog.Protocol_PROTOCOL_TCP,
VirtualPort: 9999,
},
{TargetPort: "mesh", Protocol: pbcatalog.Protocol_PROTOCOL_MESH},
},
VirtualIps: []string{"240.0.0.1"},
}).Write(t, resourceClient)
workloadPortMap := map[string]*pbcatalog.WorkloadPort{
"tcp": {
Port: 8080, Protocol: pbcatalog.Protocol_PROTOCOL_TCP,
},
"echo": {
Port: 8078, Protocol: pbcatalog.Protocol_PROTOCOL_TCP,
},
"mesh": {
Port: 20000, Protocol: pbcatalog.Protocol_PROTOCOL_MESH,
},
}
rtest.ResourceID(&pbresource.ID{
Name: "static-server-workload",
Type: pbcatalog.WorkloadType,
}).
WithData(t, &pbcatalog.Workload{
Addresses: []*pbcatalog.WorkloadAddress{
{Host: node.GetIP()},
},
Ports: workloadPortMap,
Identity: staticServerIdentity,
}).
Write(t, resourceClient)
rtest.ResourceID(&pbresource.ID{
Name: staticServerIdentity,
Type: pbauth.WorkloadIdentityType,
}).
Write(t, resourceClient)
serverDataplane, err := createServiceAndDataplane(t, node, cluster, "static-server-workload", "static-server", 8080, 8079, []int{})
require.NoError(t, err)
return serverDataplane
}
func createClientResources(t *testing.T, resourceClient *rtest.Client, cluster *libcluster.Cluster, node libcluster.Agent, idx int) *libcluster.ConsulDataplaneContainer {
prefix := fmt.Sprintf("static-client-%d", idx)
rtest.ResourceID(&pbresource.ID{
Name: prefix + "-service",
Type: pbcatalog.ServiceType,
}).
WithData(t, &pbcatalog.Service{
Workloads: &pbcatalog.WorkloadSelector{Prefixes: []string{prefix}},
Ports: []*pbcatalog.ServicePort{
{TargetPort: "tcp", Protocol: pbcatalog.Protocol_PROTOCOL_TCP},
{TargetPort: "mesh", Protocol: pbcatalog.Protocol_PROTOCOL_MESH},
},
}).Write(t, resourceClient)
workloadPortMap := map[string]*pbcatalog.WorkloadPort{
"tcp": {
Port: 8080, Protocol: pbcatalog.Protocol_PROTOCOL_TCP,
},
"mesh": {
Port: 20000, Protocol: pbcatalog.Protocol_PROTOCOL_MESH,
},
}
rtest.ResourceID(&pbresource.ID{
Name: prefix + "-workload",
Type: pbcatalog.WorkloadType,
}).
WithData(t, &pbcatalog.Workload{
Addresses: []*pbcatalog.WorkloadAddress{
{Host: node.GetIP()},
},
Ports: workloadPortMap,
Identity: prefix + "-identity",
}).
Write(t, resourceClient)
rtest.ResourceID(&pbresource.ID{
Name: prefix + "-identity",
Type: pbauth.WorkloadIdentityType,
}).
Write(t, resourceClient)
rtest.ResourceID(&pbresource.ID{
Name: prefix + "-proxy-configuration",
Type: pbmesh.ProxyConfigurationType,
}).
WithData(t, &pbmesh.ProxyConfiguration{
Workloads: &pbcatalog.WorkloadSelector{
Prefixes: []string{"static-client"},
},
DynamicConfig: &pbmesh.DynamicConfig{
Mode: pbmesh.ProxyMode_PROXY_MODE_TRANSPARENT,
},
}).
Write(t, resourceClient)
dp, err := createServiceAndDataplane(t, node, cluster, fmt.Sprintf("static-client-%d-workload", idx), "static-client", 8080, 8079, []int{})
require.NoError(t, err)
return dp
}
func createCluster(t *testing.T, aclsEnabled bool) (*libcluster.Cluster, *rtest.Client) {
cluster, _, _ := topology.NewCluster(t, &topology.ClusterConfig{
NumServers: 1,
NumClients: 3,
BuildOpts: &libcluster.BuildOptions{
Datacenter: "dc1",
InjectAutoEncryption: true,
InjectGossipEncryption: true,
AllowHTTPAnyway: true,
ACLEnabled: aclsEnabled,
},
Cmd: `-hcl=experiments=["resource-apis"] log_level="TRACE"`,
})
leader, err := cluster.Leader()
require.NoError(t, err)
client := pbresource.NewResourceServiceClient(leader.GetGRPCConn())
resourceClient := rtest.NewClientWithACLToken(client, cluster.TokenBootstrap)
return cluster, resourceClient
}
// assertDataplaneContainerState validates service container status
func assertDataplaneContainerState(t *testing.T, dataplane *libcluster.ConsulDataplaneContainer, state string) {
containerStatus, err := dataplane.GetStatus()
require.NoError(t, err)
require.Equal(t, containerStatus, state, fmt.Sprintf("Expected: %s. Got %s", state, containerStatus))
}
func httpRequestToVirtualAddress(dp *libcluster.ConsulDataplaneContainer) (string, error) {
addr := fmt.Sprintf("%s:%d", staticServerVIP, tcpPort)
out, err := dp.Exec(
context.Background(),
[]string{"sudo", "sh", "-c", fmt.Sprintf(`
set -e
curl -s "%s/debug?env=dump"
`, addr),
},
)
if err != nil {
return out, fmt.Errorf("curl request to upstream virtual address %q\nerr = %v\nout = %s\nservice=%s", addr, err, out, dp.GetServiceName())
}
expected := fmt.Sprintf("FORTIO_NAME=%s", staticServerReturnValue)
if !strings.Contains(out, expected) {
return out, fmt.Errorf("expected %q to contain %q", out, expected)
}
return out, nil
}
func echoToVirtualAddress(dp *libcluster.ConsulDataplaneContainer) (string, error) {
out, err := dp.Exec(
context.Background(),
[]string{"sudo", "sh", "-c", fmt.Sprintf(`
set -e
echo foo | nc %s %d
`, staticServerVIP, echoPort),
},
)
if err != nil {
return out, fmt.Errorf("nc request to upstream virtual address %s:%d\nerr = %v\nout = %s\nservice=%s", staticServerVIP, echoPort, err, out, dp.GetServiceName())
}
if !strings.Contains(out, "foo") {
return out, fmt.Errorf("expected %q to contain 'foo'", out)
}
return out, err
}
func assertPassing(t *retry.R, fn func(*libcluster.ConsulDataplaneContainer) (string, error), dp *libcluster.ConsulDataplaneContainer, success bool) {
_, err := fn(dp)
if success {
require.NoError(t, err)
} else {
require.Error(t, err)
}
}
Loading…
Cancel
Save